[ 
https://issues.apache.org/jira/browse/SPARK-26239?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16718068#comment-16718068
 ] 

ASF GitHub Bot commented on SPARK-26239:
----------------------------------------

asfgit closed pull request #23252: [SPARK-26239] File-based secret key loading 
for SASL.
URL: https://github.com/apache/spark/pull/23252
 
 
   

This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:

As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):

diff --git a/core/src/main/scala/org/apache/spark/SecurityManager.scala 
b/core/src/main/scala/org/apache/spark/SecurityManager.scala
index 96e4b53b24181..15783c952c231 100644
--- a/core/src/main/scala/org/apache/spark/SecurityManager.scala
+++ b/core/src/main/scala/org/apache/spark/SecurityManager.scala
@@ -17,8 +17,11 @@
 
 package org.apache.spark
 
+import java.io.File
 import java.net.{Authenticator, PasswordAuthentication}
 import java.nio.charset.StandardCharsets.UTF_8
+import java.nio.file.Files
+import java.util.Base64
 
 import org.apache.hadoop.io.Text
 import org.apache.hadoop.security.{Credentials, UserGroupInformation}
@@ -43,7 +46,8 @@ import org.apache.spark.util.Utils
  */
 private[spark] class SecurityManager(
     sparkConf: SparkConf,
-    val ioEncryptionKey: Option[Array[Byte]] = None)
+    val ioEncryptionKey: Option[Array[Byte]] = None,
+    authSecretFileConf: ConfigEntry[Option[String]] = AUTH_SECRET_FILE)
   extends Logging with SecretKeyHolder {
 
   import SecurityManager._
@@ -328,6 +332,7 @@ private[spark] class SecurityManager(
         .orElse(Option(secretKey))
         .orElse(Option(sparkConf.getenv(ENV_AUTH_SECRET)))
         .orElse(sparkConf.getOption(SPARK_AUTH_SECRET_CONF))
+        .orElse(secretKeyFromFile())
         .getOrElse {
           throw new IllegalArgumentException(
             s"A secret key must be specified via the $SPARK_AUTH_SECRET_CONF 
config")
@@ -348,7 +353,6 @@ private[spark] class SecurityManager(
    */
   def initializeAuth(): Unit = {
     import SparkMasterRegex._
-    val k8sRegex = "k8s.*".r
 
     if (!sparkConf.get(NETWORK_AUTH_ENABLED)) {
       return
@@ -371,7 +375,14 @@ private[spark] class SecurityManager(
         return
     }
 
-    secretKey = Utils.createSecret(sparkConf)
+    if (sparkConf.get(AUTH_SECRET_FILE_DRIVER).isDefined !=
+        sparkConf.get(AUTH_SECRET_FILE_EXECUTOR).isDefined) {
+      throw new IllegalArgumentException(
+        "Invalid secret configuration: Secret files must be specified for both 
the driver and the" +
+          " executors, not only one or the other.")
+    }
+
+    secretKey = secretKeyFromFile().getOrElse(Utils.createSecret(sparkConf))
 
     if (storeInUgi) {
       val creds = new Credentials()
@@ -380,6 +391,22 @@ private[spark] class SecurityManager(
     }
   }
 
+  private def secretKeyFromFile(): Option[String] = {
+    sparkConf.get(authSecretFileConf).flatMap { secretFilePath =>
+      sparkConf.getOption(SparkLauncher.SPARK_MASTER).map {
+        case k8sRegex() =>
+          val secretFile = new File(secretFilePath)
+          require(secretFile.isFile, s"No file found containing the secret key 
at $secretFilePath.")
+          val base64Key = 
Base64.getEncoder.encodeToString(Files.readAllBytes(secretFile.toPath))
+          require(!base64Key.isEmpty, s"Secret key from file located at 
$secretFilePath is empty.")
+          base64Key
+        case _ =>
+          throw new IllegalArgumentException(
+            "Secret keys provided via files is only allowed in Kubernetes 
mode.")
+      }
+    }
+  }
+
   // Default SecurityManager only has a single secret key, so ignore appId.
   override def getSaslUser(appId: String): String = getSaslUser()
   override def getSecretKey(appId: String): String = getSecretKey()
@@ -387,6 +414,7 @@ private[spark] class SecurityManager(
 
 private[spark] object SecurityManager {
 
+  val k8sRegex = "k8s.*".r
   val SPARK_AUTH_CONF = NETWORK_AUTH_ENABLED.key
   val SPARK_AUTH_SECRET_CONF = "spark.authenticate.secret"
   // This is used to set auth secret to an executor's env variable. It should 
have the same
diff --git a/core/src/main/scala/org/apache/spark/SparkEnv.scala 
b/core/src/main/scala/org/apache/spark/SparkEnv.scala
index 66038eeaea54f..de0c8579d9acc 100644
--- a/core/src/main/scala/org/apache/spark/SparkEnv.scala
+++ b/core/src/main/scala/org/apache/spark/SparkEnv.scala
@@ -232,8 +232,8 @@ object SparkEnv extends Logging {
     if (isDriver) {
       assert(listenerBus != null, "Attempted to create driver SparkEnv with 
null listener bus!")
     }
-
-    val securityManager = new SecurityManager(conf, ioEncryptionKey)
+    val authSecretFileConf = if (isDriver) AUTH_SECRET_FILE_DRIVER else 
AUTH_SECRET_FILE_EXECUTOR
+    val securityManager = new SecurityManager(conf, ioEncryptionKey, 
authSecretFileConf)
     if (isDriver) {
       securityManager.initializeAuth()
     }
diff --git a/core/src/main/scala/org/apache/spark/internal/config/package.scala 
b/core/src/main/scala/org/apache/spark/internal/config/package.scala
index 646b3881a79b0..cd57fc41c9b8e 100644
--- a/core/src/main/scala/org/apache/spark/internal/config/package.scala
+++ b/core/src/main/scala/org/apache/spark/internal/config/package.scala
@@ -419,6 +419,37 @@ package object config {
       .booleanConf
       .createWithDefault(false)
 
+  private[spark] val AUTH_SECRET_FILE =
+    ConfigBuilder("spark.authenticate.secret.file")
+      .doc("Path to a file that contains the authentication secret to use. The 
secret key is " +
+        "loaded from this path on both the driver and the executors if 
overrides are not set for " +
+        "either entity (see below). File-based secret keys are only allowed 
when using " +
+        "Kubernetes.")
+      .stringConf
+      .createOptional
+
+  private[spark] val AUTH_SECRET_FILE_DRIVER =
+    ConfigBuilder("spark.authenticate.secret.driver.file")
+      .doc("Path to a file that contains the authentication secret to use. 
Loaded by the " +
+        "driver. In Kubernetes client mode it is often useful to set a 
different secret " +
+        "path for the driver vs. the executors, since the driver may not be 
running in " +
+        "a pod unlike the executors. If this is set, an accompanying secret 
file must " +
+        "be specified for the executors. The fallback configuration allows the 
same path to be " +
+        "used for both the driver and the executors when running in cluster 
mode. File-based " +
+        "secret keys are only allowed when using Kubernetes.")
+      .fallbackConf(AUTH_SECRET_FILE)
+
+  private[spark] val AUTH_SECRET_FILE_EXECUTOR =
+    ConfigBuilder("spark.authenticate.secret.executor.file")
+      .doc("Path to a file that contains the authentication secret to use. 
Loaded by the " +
+        "executors only. In Kubernetes client mode it is often useful to set a 
different " +
+        "secret path for the driver vs. the executors, since the driver may 
not be running " +
+        "in a pod unlike the executors. If this is set, an accompanying secret 
file must be " +
+        "specified for the executors. The fallback configuration allows the 
same path to be " +
+        "used for both the driver and the executors when running in cluster 
mode. File-based " +
+        "secret keys are only allowed when using Kubernetes.")
+      .fallbackConf(AUTH_SECRET_FILE)
+
   private[spark] val NETWORK_ENCRYPTION_ENABLED =
     ConfigBuilder("spark.network.crypto.enabled")
       .booleanConf
diff --git a/core/src/test/scala/org/apache/spark/SecurityManagerSuite.scala 
b/core/src/test/scala/org/apache/spark/SecurityManagerSuite.scala
index eec8004fc94f4..e9061f4e7beb8 100644
--- a/core/src/test/scala/org/apache/spark/SecurityManagerSuite.scala
+++ b/core/src/test/scala/org/apache/spark/SecurityManagerSuite.scala
@@ -19,7 +19,9 @@ package org.apache.spark
 
 import java.io.File
 import java.nio.charset.StandardCharsets.UTF_8
+import java.nio.file.Files
 import java.security.PrivilegedExceptionAction
+import java.util.Base64
 
 import org.apache.hadoop.security.UserGroupInformation
 
@@ -395,9 +397,54 @@ class SecurityManagerSuite extends SparkFunSuite with 
ResetSystemProperties {
     assert(keyFromEnv === new SecurityManager(conf2).getSecretKey())
   }
 
+  test("use executor-specific secret file configuration.") {
+    val secretFileFromDriver = createTempSecretFile("driver-secret")
+    val secretFileFromExecutor = createTempSecretFile("executor-secret")
+    val conf = new SparkConf()
+      .setMaster("k8s://127.0.0.1")
+      .set(AUTH_SECRET_FILE_DRIVER, Some(secretFileFromDriver.getAbsolutePath))
+      .set(AUTH_SECRET_FILE_EXECUTOR, 
Some(secretFileFromExecutor.getAbsolutePath))
+      .set(SecurityManager.SPARK_AUTH_CONF, "true")
+    val mgr = new SecurityManager(conf, authSecretFileConf = 
AUTH_SECRET_FILE_EXECUTOR)
+    assert(encodeFileAsBase64(secretFileFromExecutor) === mgr.getSecretKey())
+  }
+
+  test("secret file must be defined in both driver and executor") {
+    val conf1 = new SparkConf()
+      .set(AUTH_SECRET_FILE_DRIVER, Some("/tmp/driver-secret.txt"))
+      .set(SecurityManager.SPARK_AUTH_CONF, "true")
+    val mgr1 = new SecurityManager(conf1)
+    intercept[IllegalArgumentException] {
+      mgr1.initializeAuth()
+    }
+
+    val conf2 = new SparkConf()
+      .set(AUTH_SECRET_FILE_EXECUTOR, Some("/tmp/executor-secret.txt"))
+      .set(SecurityManager.SPARK_AUTH_CONF, "true")
+    val mgr2 = new SecurityManager(conf2)
+    intercept[IllegalArgumentException] {
+      mgr2.initializeAuth()
+    }
+  }
+
+  Seq("yarn", "local", "local[*]", "local[1,2]", 
"mesos://localhost:8080").foreach { master =>
+    test(s"master $master cannot use file mounted secrets") {
+      val conf = new SparkConf()
+        .set(AUTH_SECRET_FILE, "/tmp/secret.txt")
+        .set(SecurityManager.SPARK_AUTH_CONF, "true")
+        .setMaster(master)
+      intercept[IllegalArgumentException] {
+        new SecurityManager(conf).getSecretKey()
+      }
+      intercept[IllegalArgumentException] {
+        new SecurityManager(conf).initializeAuth()
+      }
+    }
+  }
+
   // How is the secret expected to be generated and stored.
   object SecretTestType extends Enumeration {
-    val MANUAL, AUTO, UGI = Value
+    val MANUAL, AUTO, UGI, FILE = Value
   }
 
   import SecretTestType._
@@ -408,6 +455,7 @@ class SecurityManagerSuite extends SparkFunSuite with 
ResetSystemProperties {
     ("local[*]", UGI),
     ("local[1, 2]", UGI),
     ("k8s://127.0.0.1", AUTO),
+    ("k8s://127.0.1.1", FILE),
     ("local-cluster[2, 1, 1024]", MANUAL),
     ("invalid", MANUAL)
   ).foreach { case (master, secretType) =>
@@ -440,6 +488,12 @@ class SecurityManagerSuite extends SparkFunSuite with 
ResetSystemProperties {
                 intercept[IllegalArgumentException] {
                   mgr.getSecretKey()
                 }
+
+              case FILE =>
+                val secretFile = createTempSecretFile()
+                conf.set(AUTH_SECRET_FILE, secretFile.getAbsolutePath)
+                mgr.initializeAuth()
+                assert(encodeFileAsBase64(secretFile) === mgr.getSecretKey())
             }
           }
         }
@@ -447,5 +501,15 @@ class SecurityManagerSuite extends SparkFunSuite with 
ResetSystemProperties {
     }
   }
 
+  private def encodeFileAsBase64(secretFile: File) = {
+    Base64.getEncoder.encodeToString(Files.readAllBytes(secretFile.toPath))
+  }
+
+  private def createTempSecretFile(contents: String = "test-secret"): File = {
+    val secretDir = Utils.createTempDir("temp-secrets")
+    val secretFile = new File(secretDir, "temp-secret.txt")
+    Files.write(secretFile.toPath, contents.getBytes(UTF_8))
+    secretFile
+  }
 }
 
diff --git a/docs/security.md b/docs/security.md
index 2a4f3c074c1e5..8416ed91356aa 100644
--- a/docs/security.md
+++ b/docs/security.md
@@ -66,6 +66,50 @@ Kubernetes admin to ensure that Spark authentication is 
secure.
 </tr>
 </table>
 
+Alternatively, one can mount authentication secrets using files and Kubernetes 
secrets that
+the user mounts into their pods.
+
+<table class="table">
+<tr><th>Property Name</th><th>Default</th><th>Meaning</th></tr>
+<tr>
+  <td><code>spark.authenticate.secret.file</code></td>
+  <td>None</td>
+  <td>
+    Path pointing to the secret key to use for securing connections. Ensure 
that the
+    contents of the file have been securely generated. This file is loaded on 
both the driver
+    and the executors unless other settings override this (see below).
+  </td>
+</tr>
+<tr>
+  <td><code>spark.authenticate.secret.driver.file</code></td>
+  <td>The value of <code>spark.authenticate.secret.file</code></td>
+  <td>
+    When specified, overrides the location that the Spark driver reads to load 
the secret.
+    Useful when in client mode, when the location of the secret file may 
differ in the pod versus
+    the node the driver is running in. When this is specified,
+    <code>spark.authenticate.secret.executor.file</code> must be specified so 
that the driver
+    and the executors can both use files to load the secret key. Ensure that 
the contents of the file
+    on the driver is identical to the contents of the file on the executors.
+  </td>
+</tr>
+<tr>
+  <td><code>spark.authenticate.secret.executor.file</code></td>
+  <td>The value of <code>spark.authenticate.secret.file</code></td>
+  <td>
+    When specified, overrides the location that the Spark executors read to 
load the secret.
+    Useful in client mode, when the location of the secret file may differ in 
the pod versus
+    the node the driver is running in. When this is specified,
+    <code>spark.authenticate.secret.driver.file</code> must be specified so 
that the driver
+    and the executors can both use files to load the secret key. Ensure that 
the contents of the file
+    on the driver is identical to the contents of the file on the executors.
+  </td>
+</tr>
+</table>
+
+Note that when using files, Spark will not mount these files into the 
containers for you. It is up
+you to ensure that the secret files are deployed securely into your containers 
and that the driver's
+secret file agrees with the executors' secret file.
+
 ## Encryption
 
 Spark supports AES-based encryption for RPC connections. For encryption to be 
enabled, RPC
diff --git 
a/resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStep.scala
 
b/resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStep.scala
index 939aa88b07973..4bcf4c9446aa3 100644
--- 
a/resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStep.scala
+++ 
b/resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStep.scala
@@ -24,7 +24,7 @@ import org.apache.spark.{SecurityManager, SparkConf, 
SparkException}
 import org.apache.spark.deploy.k8s._
 import org.apache.spark.deploy.k8s.Config._
 import org.apache.spark.deploy.k8s.Constants._
-import org.apache.spark.internal.config.{EXECUTOR_CLASS_PATH, 
EXECUTOR_JAVA_OPTIONS, EXECUTOR_MEMORY, EXECUTOR_MEMORY_OVERHEAD, 
PYSPARK_EXECUTOR_MEMORY}
+import org.apache.spark.internal.config._
 import org.apache.spark.rpc.RpcEndpointAddress
 import org.apache.spark.scheduler.cluster.CoarseGrainedSchedulerBackend
 import org.apache.spark.util.Utils
@@ -112,12 +112,14 @@ private[spark] class BasicExecutorFeatureStep(
             .build())
           .build())
       } ++ {
-        Option(secMgr.getSecretKey()).map { authSecret =>
-          new EnvVarBuilder()
-            .withName(SecurityManager.ENV_AUTH_SECRET)
-            .withValue(authSecret)
-            .build()
-        }
+        if (kubernetesConf.get(AUTH_SECRET_FILE_EXECUTOR).isEmpty) {
+          Option(secMgr.getSecretKey()).map { authSecret =>
+            new EnvVarBuilder()
+              .withName(SecurityManager.ENV_AUTH_SECRET)
+              .withValue(authSecret)
+              .build()
+          }
+        } else None
       } ++ {
         kubernetesConf.get(EXECUTOR_CLASS_PATH).map { cp =>
           new EnvVarBuilder()
diff --git 
a/resource-managers/kubernetes/core/src/test/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStepSuite.scala
 
b/resource-managers/kubernetes/core/src/test/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStepSuite.scala
index 6aa862643c788..05989d9be7ad5 100644
--- 
a/resource-managers/kubernetes/core/src/test/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStepSuite.scala
+++ 
b/resource-managers/kubernetes/core/src/test/scala/org/apache/spark/deploy/k8s/features/BasicExecutorFeatureStepSuite.scala
@@ -16,6 +16,10 @@
  */
 package org.apache.spark.deploy.k8s.features
 
+import java.io.File
+import java.nio.charset.StandardCharsets
+import java.nio.file.Files
+
 import scala.collection.JavaConverters._
 
 import io.fabric8.kubernetes.api.model._
@@ -158,6 +162,25 @@ class BasicExecutorFeatureStepSuite extends SparkFunSuite 
with BeforeAndAfter {
     checkEnv(executor, conf, Map(SecurityManager.ENV_AUTH_SECRET -> 
secMgr.getSecretKey()))
   }
 
+  test("Auth secret shouldn't propagate if files are loaded.") {
+    val secretDir = Utils.createTempDir("temp-secret")
+    val secretFile = new File(secretDir, "secret-file.txt")
+    Files.write(secretFile.toPath, 
"some-secret".getBytes(StandardCharsets.UTF_8))
+    val conf = baseConf.clone()
+      .set(NETWORK_AUTH_ENABLED, true)
+      .set(AUTH_SECRET_FILE, secretFile.getAbsolutePath)
+      .set("spark.master", "k8s://127.0.0.1")
+    val secMgr = new SecurityManager(conf)
+    secMgr.initializeAuth()
+
+    val step = new 
BasicExecutorFeatureStep(KubernetesTestConf.createExecutorConf(sparkConf = 
conf),
+      secMgr)
+
+    val executor = step.configurePod(SparkPod.initialPod())
+    assert(!KubernetesFeaturesTestUtils.containerHasEnvVar(
+      executor.container, SecurityManager.ENV_AUTH_SECRET))
+  }
+
   // There is always exactly one controller reference, and it points to the 
driver pod.
   private def checkOwnerReferences(executor: Pod, driverPodUid: String): Unit 
= {
     assert(executor.getMetadata.getOwnerReferences.size() === 1)


 

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


> Add configurable auth secret source in k8s backend
> --------------------------------------------------
>
>                 Key: SPARK-26239
>                 URL: https://issues.apache.org/jira/browse/SPARK-26239
>             Project: Spark
>          Issue Type: New Feature
>          Components: Kubernetes
>    Affects Versions: 3.0.0
>            Reporter: Marcelo Vanzin
>            Assignee: Matt Cheah
>            Priority: Major
>             Fix For: 3.0.0
>
>
> This is a follow up to SPARK-26194, which aims to add auto-generated secrets 
> similar to the YARN backend.
> There's a desire to support different ways to generate and propagate these 
> auth secrets (e.g. using things like Vault). Need to investigate:
> - exposing configuration to support that
> - changing SecurityManager so that it can delegate some of the 
> secret-handling logic to custom implementations
> - figuring out whether this can also be used in client-mode, where the driver 
> is not created by the k8s backend in Spark.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscr...@spark.apache.org
For additional commands, e-mail: issues-h...@spark.apache.org

Reply via email to