Skip to content
This repository was archived by the owner on Jan 9, 2020. It is now read-only.

Commit e84737d

Browse files
committed
Download remotely-located resources on driver startup.
Augments the init-container so that we don't need to use a separate image, but on submission two containers are bootstrapped instead for a cleaner architecture.
1 parent 04afcf8 commit e84737d

26 files changed

+1153
-284
lines changed

resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/config.scala

Lines changed: 38 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -365,7 +365,7 @@ package object config extends Logging {
365365
" resource staging server to download jars.")
366366
.internal()
367367
.stringConf
368-
.createWithDefault(INIT_CONTAINER_DOWNLOAD_JARS_SECRET_PATH)
368+
.createWithDefault(INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_JARS_SECRET_PATH)
369369

370370
private[spark] val INIT_CONTAINER_DOWNLOAD_FILES_RESOURCE_IDENTIFIER =
371371
ConfigBuilder("spark.kubernetes.driver.initcontainer.downloadFilesResourceIdentifier")
@@ -380,30 +380,62 @@ package object config extends Logging {
380380
" resource staging server to download files.")
381381
.internal()
382382
.stringConf
383-
.createWithDefault(INIT_CONTAINER_DOWNLOAD_FILES_SECRET_PATH)
383+
.createWithDefault(INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_FILES_SECRET_PATH)
384+
385+
private[spark] val INIT_CONTAINER_REMOTE_JARS =
386+
ConfigBuilder("spark.kubernetes.driver.initcontainer.remoteJars")
387+
.doc("Comma-separated list of jar URIs to download in the init-container. This is inferred" +
388+
" from spark.jars.")
389+
.internal()
390+
.stringConf
391+
.createOptional
392+
393+
private[spark] val INIT_CONTAINER_REMOTE_FILES =
394+
ConfigBuilder("spark.kubernetes.driver.initcontainer.remoteFiles")
395+
.doc("Comma-separated list of file URIs to download in the init-container. This is inferred" +
396+
" from spark.files.")
397+
.internal()
398+
.stringConf
399+
.createOptional
384400

385401
private[spark] val INIT_CONTAINER_DOCKER_IMAGE =
386402
ConfigBuilder("spark.kubernetes.driver.initcontainer.docker.image")
387403
.doc("Image for the driver's init-container that downloads mounted dependencies.")
388404
.stringConf
389405
.createWithDefault(s"spark-driver-init:$sparkVersion")
390406

391-
private[spark] val DRIVER_LOCAL_JARS_DOWNLOAD_LOCATION =
392-
ConfigBuilder("spark.kubernetes.driver.mountdependencies.jarsDownloadDir")
407+
private[spark] val DRIVER_SUBMITTED_JARS_DOWNLOAD_LOCATION =
408+
ConfigBuilder("spark.kubernetes.driver.mountdependencies.submittedJars.downloadDir")
393409
.doc("Location to download local jars to in the driver. When using spark-submit, this" +
394410
" directory must be empty and will be mounted as an empty directory volume on the" +
395411
" driver pod.")
396412
.stringConf
397413
.createWithDefault("/var/spark-data/spark-local-jars")
398414

399-
private[spark] val DRIVER_LOCAL_FILES_DOWNLOAD_LOCATION =
400-
ConfigBuilder("spark.kubernetes.driver.mountdependencies.filesDownloadDir")
415+
private[spark] val DRIVER_SUBMITTED_FILES_DOWNLOAD_LOCATION =
416+
ConfigBuilder("spark.kubernetes.driver.mountdependencies.submittedFiles.downloadDir")
401417
.doc("Location to download local files to in the driver. When using spark-submit, this" +
402418
" directory must be empty and will be mounted as an empty directory volume on the" +
403419
" driver pod.")
404420
.stringConf
405421
.createWithDefault("/var/spark-data/spark-local-files")
406422

423+
private[spark] val DRIVER_REMOTE_JARS_DOWNLOAD_LOCATION =
424+
ConfigBuilder("spark.kubernetes.driver.mountdependencies.remoteJars.downloadDir")
425+
.doc("Location to download remotely-located (e.g. HDFS) jars to in the driver. When" +
426+
" using spark-submit, this directory must be empty and will be mounted as an empty" +
427+
" directory volume on the driver pod.")
428+
.stringConf
429+
.createWithDefault("/var/spark-data/spark-remote-jars")
430+
431+
private[spark] val DRIVER_REMOTE_FILES_DOWNLOAD_LOCATION =
432+
ConfigBuilder("spark.kubernetes.driver.mountdependencies.remoteFiles.downloadDir")
433+
.doc("Location to download remotely-located (e.g. HDFS) files to in the driver. When" +
434+
" using spark-submit, this directory must be empty and will be mounted as an empty" +
435+
" directory volume on the driver pod.")
436+
.stringConf
437+
.createWithDefault("/var/spark-data/spark-remote-files")
438+
407439
private[spark] val DRIVER_MOUNT_DEPENDENCIES_INIT_TIMEOUT =
408440
ConfigBuilder("spark.kubernetes.mountdependencies.mountTimeout")
409441
.doc("Timeout before aborting the attempt to download and unpack local dependencies from" +

resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/constants.scala

Lines changed: 55 additions & 22 deletions
Original file line numberDiff line numberDiff line change
@@ -70,7 +70,6 @@ package object constants {
7070
private[spark] val ENV_EXECUTOR_ID = "SPARK_EXECUTOR_ID"
7171
private[spark] val ENV_EXECUTOR_POD_IP = "SPARK_EXECUTOR_POD_IP"
7272
private[spark] val ENV_DRIVER_MEMORY = "SPARK_DRIVER_MEMORY"
73-
private[spark] val ENV_UPLOADED_JARS_DIR = "SPARK_UPLOADED_JARS_DIR"
7473
private[spark] val ENV_SUBMIT_EXTRA_CLASSPATH = "SPARK_SUBMIT_EXTRA_CLASSPATH"
7574
private[spark] val ENV_MOUNTED_CLASSPATH = "SPARK_MOUNTED_CLASSPATH"
7675
private[spark] val ENV_DRIVER_MAIN_CLASS = "SPARK_DRIVER_CLASS"
@@ -92,25 +91,59 @@ package object constants {
9291

9392
// V2 submission init container
9493
private[spark] val INIT_CONTAINER_ANNOTATION = "pod.beta.kubernetes.io/init-containers"
95-
private[spark] val INIT_CONTAINER_SECRETS_VOLUME_NAME = "dependency-secret"
96-
private[spark] val INIT_CONTAINER_SECRETS_VOLUME_MOUNT_PATH = "/mnt/secrets/spark-init"
97-
private[spark] val INIT_CONTAINER_DOWNLOAD_JARS_SECRET_KEY = "downloadJarsSecret"
98-
private[spark] val INIT_CONTAINER_DOWNLOAD_FILES_SECRET_KEY = "downloadFilesSecret"
99-
private[spark] val INIT_CONTAINER_TRUSTSTORE_SECRET_KEY = "trustStore"
100-
private[spark] val INIT_CONTAINER_DOWNLOAD_JARS_SECRET_PATH =
101-
s"$INIT_CONTAINER_SECRETS_VOLUME_MOUNT_PATH/$INIT_CONTAINER_DOWNLOAD_JARS_SECRET_KEY"
102-
private[spark] val INIT_CONTAINER_DOWNLOAD_FILES_SECRET_PATH =
103-
s"$INIT_CONTAINER_SECRETS_VOLUME_MOUNT_PATH/$INIT_CONTAINER_DOWNLOAD_FILES_SECRET_KEY"
104-
private[spark] val INIT_CONTAINER_TRUSTSTORE_PATH =
105-
s"$INIT_CONTAINER_SECRETS_VOLUME_MOUNT_PATH/$INIT_CONTAINER_TRUSTSTORE_SECRET_KEY"
106-
private[spark] val INIT_CONTAINER_DOWNLOAD_CREDENTIALS_PATH =
107-
"/mnt/secrets/kubernetes-credentials"
108-
private[spark] val INIT_CONTAINER_CONFIG_MAP_KEY = "init-driver"
109-
private[spark] val INIT_CONTAINER_PROPERTIES_FILE_VOLUME = "init-container-properties"
110-
private[spark] val INIT_CONTAINER_PROPERTIES_FILE_MOUNT_PATH = "/etc/spark-init/"
111-
private[spark] val INIT_CONTAINER_PROPERTIES_FILE_NAME = "init-driver.properties"
112-
private[spark] val INIT_CONTAINER_PROPERTIES_FILE_PATH =
113-
s"$INIT_CONTAINER_PROPERTIES_FILE_MOUNT_PATH/$INIT_CONTAINER_PROPERTIES_FILE_NAME"
114-
private[spark] val DOWNLOAD_JARS_VOLUME_NAME = "download-jars"
115-
private[spark] val DOWNLOAD_FILES_VOLUME_NAME = "download-files"
94+
95+
// Init container for downloading submitted files from the staging server.
96+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_CONTAINER_NAME =
97+
"spark-driver-download-submitted-files"
98+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_SECRETS_VOLUME_NAME =
99+
"resource-staging-server-secret"
100+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_SECRETS_VOLUME_MOUNT_PATH =
101+
"/mnt/secrets/spark-init"
102+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_JARS_SECRET_KEY =
103+
"downloadSubmittedJarsSecret"
104+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_FILES_SECRET_KEY =
105+
"downloadSubmittedFilesSecret"
106+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_TRUSTSTORE_SECRET_KEY = "trustStore"
107+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_JARS_SECRET_PATH =
108+
s"$INIT_CONTAINER_SUBMITTED_FILES_SECRETS_VOLUME_MOUNT_PATH/" +
109+
s"$INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_JARS_SECRET_KEY"
110+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_FILES_SECRET_PATH =
111+
s"$INIT_CONTAINER_SUBMITTED_FILES_SECRETS_VOLUME_MOUNT_PATH/" +
112+
s"$INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_FILES_SECRET_KEY"
113+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_TRUSTSTORE_PATH =
114+
s"$INIT_CONTAINER_SUBMITTED_FILES_SECRETS_VOLUME_MOUNT_PATH/" +
115+
s"$INIT_CONTAINER_SUBMITTED_FILES_TRUSTSTORE_SECRET_KEY"
116+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_CONFIG_MAP_KEY =
117+
"download-submitted-files"
118+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_PROPERTIES_FILE_VOLUME =
119+
"download-submitted-files-properties"
120+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_PROPERTIES_FILE_MOUNT_PATH =
121+
"/etc/spark-init/download-submitted-files"
122+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_PROPERTIES_FILE_NAME =
123+
"init-driver-download-submitted-files.properties"
124+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_PROPERTIES_FILE_PATH =
125+
s"$INIT_CONTAINER_SUBMITTED_FILES_PROPERTIES_FILE_MOUNT_PATH/" +
126+
s"$INIT_CONTAINER_SUBMITTED_FILES_PROPERTIES_FILE_NAME"
127+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_JARS_VOLUME_NAME =
128+
"download-submitted-jars"
129+
private[spark] val INIT_CONTAINER_SUBMITTED_FILES_DOWNLOAD_FILES_VOLUME_NAME =
130+
"download-submitted-files"
131+
132+
// Init container for fetching remote dependencies.
133+
private[spark] val INIT_CONTAINER_REMOTE_FILES_CONTAINER_NAME =
134+
"spark-driver-download-remote-files"
135+
private[spark] val INIT_CONTAINER_REMOTE_FILES_CONFIG_MAP_KEY =
136+
"download-remote-files"
137+
private[spark] val INIT_CONTAINER_REMOTE_FILES_PROPERTIES_FILE_VOLUME =
138+
"download-remote-files-properties"
139+
private[spark] val INIT_CONTAINER_REMOTE_FILES_PROPERTIES_FILE_MOUNT_PATH =
140+
"/etc/spark-init/download-remote-files"
141+
private[spark] val INIT_CONTAINER_REMOTE_FILES_PROPERTIES_FILE_NAME =
142+
"init-driver-download-remote-files.properties"
143+
private[spark] val INIT_CONTAINER_REMOTE_FILES_PROPERTIES_FILE_PATH =
144+
s"$INIT_CONTAINER_REMOTE_FILES_PROPERTIES_FILE_MOUNT_PATH/" +
145+
s"$INIT_CONTAINER_REMOTE_FILES_PROPERTIES_FILE_NAME"
146+
private[spark] val INIT_CONTAINER_REMOTE_FILES_DOWNLOAD_JARS_VOLUME_NAME = "download-remote-jars"
147+
private[spark] val INIT_CONTAINER_REMOTE_FILES_DOWNLOAD_FILES_VOLUME_NAME =
148+
"download-remote-files"
116149
}
Lines changed: 5 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -14,7 +14,7 @@
1414
* See the License for the specific language governing permissions and
1515
* limitations under the License.
1616
*/
17-
package org.apache.spark.deploy.rest.kubernetes.v1
17+
package org.apache.spark.deploy.kubernetes.submit
1818

1919
import org.apache.spark.util.Utils
2020

@@ -41,4 +41,8 @@ private[spark] object KubernetesFileUtils {
4141
Option(Utils.resolveURI(uri).getScheme).getOrElse("file") == "file"
4242
}
4343

44+
def getOnlyRemoteFiles(uris: Iterable[String]): Iterable[String] = {
45+
filterUriStringsByScheme(uris, scheme => scheme != "file" && scheme != "local")
46+
}
47+
4448
}

resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/submit/v1/Client.scala

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -33,7 +33,8 @@ import org.apache.spark.{SparkConf, SparkException}
3333
import org.apache.spark.deploy.kubernetes.CompressionUtils
3434
import org.apache.spark.deploy.kubernetes.config._
3535
import org.apache.spark.deploy.kubernetes.constants._
36-
import org.apache.spark.deploy.rest.kubernetes.v1.{AppResource, ContainerAppResource, HttpClientUtil, KubernetesCreateSubmissionRequest, KubernetesCredentials, KubernetesFileUtils, KubernetesSparkRestApi, RemoteAppResource, UploadedAppResource}
36+
import org.apache.spark.deploy.kubernetes.submit.KubernetesFileUtils
37+
import org.apache.spark.deploy.rest.kubernetes.v1.{AppResource, ContainerAppResource, HttpClientUtil, KubernetesCreateSubmissionRequest, KubernetesCredentials, KubernetesSparkRestApi, RemoteAppResource, UploadedAppResource}
3738
import org.apache.spark.internal.Logging
3839
import org.apache.spark.util.{ShutdownHookManager, Utils}
3940

resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/submit/v1/DriverSubmitSslConfigurationProvider.scala

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -29,7 +29,8 @@ import scala.collection.JavaConverters._
2929
import org.apache.spark.{SecurityManager => SparkSecurityManager, SparkConf, SparkException, SSLOptions}
3030
import org.apache.spark.deploy.kubernetes.config._
3131
import org.apache.spark.deploy.kubernetes.constants._
32-
import org.apache.spark.deploy.rest.kubernetes.v1.{KubernetesFileUtils, PemsToKeyStoreConverter}
32+
import org.apache.spark.deploy.kubernetes.submit.KubernetesFileUtils
33+
import org.apache.spark.deploy.rest.kubernetes.v1.PemsToKeyStoreConverter
3334
import org.apache.spark.util.Utils
3435

3536
/**

resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/kubernetes/submit/v2/Client.scala

Lines changed: 46 additions & 22 deletions
Original file line numberDiff line numberDiff line change
@@ -17,13 +17,14 @@
1717
package org.apache.spark.deploy.kubernetes.submit.v2
1818

1919
import java.io.File
20+
import java.net.URI
2021
import java.util.Collections
2122

2223
import io.fabric8.kubernetes.api.model.{ContainerBuilder, EnvVarBuilder, HasMetadata, OwnerReferenceBuilder, PodBuilder}
2324
import scala.collection.JavaConverters._
2425
import scala.collection.mutable
2526

26-
import org.apache.spark.{SecurityManager => SparkSecurityManager, SparkConf, SparkException}
27+
import org.apache.spark.{SparkConf, SparkException}
2728
import org.apache.spark.deploy.kubernetes.config._
2829
import org.apache.spark.deploy.kubernetes.constants._
2930
import org.apache.spark.internal.Logging
@@ -50,7 +51,8 @@ private[spark] class Client(
5051
appArgs: Array[String],
5152
mainAppResource: String,
5253
kubernetesClientProvider: SubmissionKubernetesClientProvider,
53-
mountedDependencyManagerProvider: MountedDependencyManagerProvider) extends Logging {
54+
submittedDependencyManagerProvider: SubmittedDependencyManagerProvider,
55+
remoteDependencyManagerProvider: DownloadRemoteDependencyManagerProvider) extends Logging {
5456

5557
private val namespace = sparkConf.get(KUBERNETES_NAMESPACE)
5658
private val master = resolveK8sMaster(sparkConf.get("spark.master"))
@@ -132,27 +134,33 @@ private[spark] class Client(
132134
.endSpec()
133135

134136
val nonDriverPodKubernetesResources = mutable.Buffer[HasMetadata]()
137+
138+
// resolvedJars is all of the original Spark jars, but files on the local disk that
139+
// were uploaded to the resource staging server are converted to the paths they would
140+
// have been downloaded at. Similarly for resolvedFiles.
141+
// If the resource staging server isn't being used, then resolvedJars = spark.jars.
135142
val resolvedJars = mutable.Buffer[String]()
136143
val resolvedFiles = mutable.Buffer[String]()
137144
val driverPodWithMountedDeps = maybeStagingServerUri.map { stagingServerUri =>
138-
val mountedDependencyManager = mountedDependencyManagerProvider.getMountedDependencyManager(
139-
kubernetesAppId,
140-
stagingServerUri,
141-
allLabels,
142-
namespace,
143-
sparkJars,
144-
sparkFiles)
145-
val jarsResourceIdentifier = mountedDependencyManager.uploadJars()
146-
val filesResourceIdentifier = mountedDependencyManager.uploadFiles()
147-
val initContainerKubernetesSecret = mountedDependencyManager.buildInitContainerSecret(
145+
val submittedDependencyManager = submittedDependencyManagerProvider
146+
.getSubmittedDependencyManager(
147+
kubernetesAppId,
148+
stagingServerUri,
149+
allLabels,
150+
namespace,
151+
sparkJars,
152+
sparkFiles)
153+
val jarsResourceIdentifier = submittedDependencyManager.uploadJars()
154+
val filesResourceIdentifier = submittedDependencyManager.uploadFiles()
155+
val initContainerKubernetesSecret = submittedDependencyManager.buildInitContainerSecret(
148156
jarsResourceIdentifier.resourceSecret, filesResourceIdentifier.resourceSecret)
149-
val initContainerConfigMap = mountedDependencyManager.buildInitContainerConfigMap(
157+
val initContainerConfigMap = submittedDependencyManager.buildInitContainerConfigMap(
150158
jarsResourceIdentifier.resourceId, filesResourceIdentifier.resourceId)
151-
resolvedJars ++= mountedDependencyManager.resolveSparkJars()
152-
resolvedFiles ++= mountedDependencyManager.resolveSparkFiles()
159+
resolvedJars ++= submittedDependencyManager.resolveSparkJars()
160+
resolvedFiles ++= submittedDependencyManager.resolveSparkFiles()
153161
nonDriverPodKubernetesResources += initContainerKubernetesSecret
154162
nonDriverPodKubernetesResources += initContainerConfigMap
155-
mountedDependencyManager.configurePodToMountLocalDependencies(
163+
submittedDependencyManager.configurePodToMountLocalDependencies(
156164
driverContainer.getName, initContainerKubernetesSecret, initContainerConfigMap, basePod)
157165
}.getOrElse {
158166
sparkJars.map(Utils.resolveURI).foreach { jar =>
@@ -186,19 +194,35 @@ private[spark] class Client(
186194
resolvedSparkConf.get(KUBERNETES_DRIVER_OAUTH_TOKEN).foreach { _ =>
187195
resolvedSparkConf.set(KUBERNETES_DRIVER_OAUTH_TOKEN.key, "<present_but_redacted>")
188196
}
197+
val remoteDependencyManager = remoteDependencyManagerProvider
198+
.getDownloadRemoteDependencyManager(
199+
kubernetesAppId,
200+
resolvedJars,
201+
resolvedFiles)
202+
val downloadRemoteDependenciesConfigMap = remoteDependencyManager
203+
.buildInitContainerConfigMap()
204+
nonDriverPodKubernetesResources += downloadRemoteDependenciesConfigMap
205+
val driverPodWithMountedAndDownloadedDeps = remoteDependencyManager
206+
.configurePodToDownloadRemoteDependencies(
207+
downloadRemoteDependenciesConfigMap, driverContainer.getName, driverPodWithMountedDeps)
189208

190-
val mountedClassPath = resolvedJars.map(Utils.resolveURI).filter { jarUri =>
191-
val scheme = Option.apply(jarUri.getScheme).getOrElse("file")
192-
scheme == "local" || scheme == "file"
193-
}.map(_.getPath).mkString(File.pathSeparator)
209+
// The resolved local classpath should *only* contain local file URIs. It consists of the
210+
// driver's classpath (minus spark.driver.extraClassPath which was handled above) with the
211+
// assumption that the remote dependency manager has downloaded all of the remote
212+
// dependencies through its init-container, and thus replaces all the remote URIs with the
213+
// local paths they were downloaded to.
214+
val resolvedLocalClassPath = remoteDependencyManager.resolveLocalClasspath()
215+
resolvedLocalClassPath.foreach { classPathEntry =>
216+
require(Option(URI.create(classPathEntry).getScheme).isEmpty)
217+
}
194218
val resolvedDriverJavaOpts = resolvedSparkConf.getAll.map { case (confKey, confValue) =>
195219
s"-D$confKey=$confValue"
196220
}.mkString(" ") + driverJavaOptions.map(" " + _).getOrElse("")
197-
val resolvedDriverPod = driverPodWithMountedDeps.editSpec()
221+
val resolvedDriverPod = driverPodWithMountedAndDownloadedDeps.editSpec()
198222
.editMatchingContainer(new ContainerNameEqualityPredicate(driverContainer.getName))
199223
.addNewEnv()
200224
.withName(ENV_MOUNTED_CLASSPATH)
201-
.withValue(mountedClassPath)
225+
.withValue(resolvedLocalClassPath.mkString(File.pathSeparator))
202226
.endEnv()
203227
.addNewEnv()
204228
.withName(ENV_DRIVER_JAVA_OPTS)

0 commit comments

Comments
 (0)