Skip to content

Commit 5d7c4ba

Browse files
liyinan926Marcelo Vanzin
authored andcommitted
[SPARK-22962][K8S] Fail fast if submission client local files are used
## What changes were proposed in this pull request? In the Kubernetes mode, fails fast in the submission process if any submission client local dependencies are used as the use case is not supported yet. ## How was this patch tested? Unit tests, integration tests, and manual tests. vanzin foxish Author: Yinan Li <[email protected]> Closes #20320 from liyinan926/master.
1 parent e01919e commit 5d7c4ba

File tree

3 files changed

+47
-3
lines changed

3 files changed

+47
-3
lines changed

docs/running-on-kubernetes.md

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -117,7 +117,10 @@ This URI is the location of the example jar that is already in the Docker image.
117117
If your application's dependencies are all hosted in remote locations like HDFS or HTTP servers, they may be referred to
118118
by their appropriate remote URIs. Also, application dependencies can be pre-mounted into custom-built Docker images.
119119
Those dependencies can be added to the classpath by referencing them with `local://` URIs and/or setting the
120-
`SPARK_EXTRA_CLASSPATH` environment variable in your Dockerfiles.
120+
`SPARK_EXTRA_CLASSPATH` environment variable in your Dockerfiles. The `local://` scheme is also required when referring to
121+
dependencies in custom-built Docker images in `spark-submit`. Note that using application dependencies from the submission
122+
client's local file system is currently not yet supported.
123+
121124

122125
### Using Remote Dependencies
123126
When there are application dependencies hosted in remote locations like HDFS or HTTP servers, the driver and executor pods

resource-managers/kubernetes/core/src/main/scala/org/apache/spark/deploy/k8s/submit/DriverConfigOrchestrator.scala

Lines changed: 13 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,7 @@ import java.util.UUID
2020

2121
import com.google.common.primitives.Longs
2222

23-
import org.apache.spark.SparkConf
23+
import org.apache.spark.{SparkConf, SparkException}
2424
import org.apache.spark.deploy.k8s.{KubernetesUtils, MountSecretsBootstrap}
2525
import org.apache.spark.deploy.k8s.Config._
2626
import org.apache.spark.deploy.k8s.Constants._
@@ -117,6 +117,12 @@ private[spark] class DriverConfigOrchestrator(
117117
.map(_.split(","))
118118
.getOrElse(Array.empty[String])
119119

120+
// TODO(SPARK-23153): remove once submission client local dependencies are supported.
121+
if (existSubmissionLocalFiles(sparkJars) || existSubmissionLocalFiles(sparkFiles)) {
122+
throw new SparkException("The Kubernetes mode does not yet support referencing application " +
123+
"dependencies in the local file system.")
124+
}
125+
120126
val dependencyResolutionStep = if (sparkJars.nonEmpty || sparkFiles.nonEmpty) {
121127
Seq(new DependencyResolutionStep(
122128
sparkJars,
@@ -162,6 +168,12 @@ private[spark] class DriverConfigOrchestrator(
162168
initContainerBootstrapStep
163169
}
164170

171+
private def existSubmissionLocalFiles(files: Seq[String]): Boolean = {
172+
files.exists { uri =>
173+
Utils.resolveURI(uri).getScheme == "file"
174+
}
175+
}
176+
165177
private def existNonContainerLocalFiles(files: Seq[String]): Boolean = {
166178
files.exists { uri =>
167179
Utils.resolveURI(uri).getScheme != "local"

resource-managers/kubernetes/core/src/test/scala/org/apache/spark/deploy/k8s/submit/DriverConfigOrchestratorSuite.scala

Lines changed: 30 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -16,7 +16,7 @@
1616
*/
1717
package org.apache.spark.deploy.k8s.submit
1818

19-
import org.apache.spark.{SparkConf, SparkFunSuite}
19+
import org.apache.spark.{SparkConf, SparkException, SparkFunSuite}
2020
import org.apache.spark.deploy.k8s.Config._
2121
import org.apache.spark.deploy.k8s.submit.steps._
2222

@@ -117,6 +117,35 @@ class DriverConfigOrchestratorSuite extends SparkFunSuite {
117117
classOf[DriverMountSecretsStep])
118118
}
119119

120+
test("Submission using client local dependencies") {
121+
val sparkConf = new SparkConf(false)
122+
.set(CONTAINER_IMAGE, DRIVER_IMAGE)
123+
var orchestrator = new DriverConfigOrchestrator(
124+
APP_ID,
125+
LAUNCH_TIME,
126+
Some(JavaMainAppResource("file:///var/apps/jars/main.jar")),
127+
APP_NAME,
128+
MAIN_CLASS,
129+
APP_ARGS,
130+
sparkConf)
131+
assertThrows[SparkException] {
132+
orchestrator.getAllConfigurationSteps
133+
}
134+
135+
sparkConf.set("spark.files", "/path/to/file1,/path/to/file2")
136+
orchestrator = new DriverConfigOrchestrator(
137+
APP_ID,
138+
LAUNCH_TIME,
139+
Some(JavaMainAppResource("local:///var/apps/jars/main.jar")),
140+
APP_NAME,
141+
MAIN_CLASS,
142+
APP_ARGS,
143+
sparkConf)
144+
assertThrows[SparkException] {
145+
orchestrator.getAllConfigurationSteps
146+
}
147+
}
148+
120149
private def validateStepTypes(
121150
orchestrator: DriverConfigOrchestrator,
122151
types: Class[_ <: DriverConfigurationStep]*): Unit = {

0 commit comments

Comments
 (0)