Skip to content

Commit 4cc8101

Browse files
committed
[SPARK-52830][K8S] Support spark.kubernetes.(driver|executor).pod.excludedFeatureSteps
### What changes were proposed in this pull request? This PR aims to support `spark.kubernetes.(driver|executor).pod.excludedFeatureSteps` configuration. ### Why are the changes needed? Since Apache Spark 3.2, we have been providing `spark.kubernetes.(driver|executor).pod.featureSteps`. - #30206 This PR aims to allow users to exclude feature steps selectively by configurations. Please note that this is designed to allow to exclude all steps including both built-in and user-provided steps. ### Does this PR introduce _any_ user-facing change? No because this is a new feature. ### How was this patch tested? Pass the CIs with newly added test cases. ### Was this patch authored or co-authored using generative AI tooling? No. Closes #51522 from dongjoon-hyun/SPARK-52830. Authored-by: Dongjoon Hyun <[email protected]> Signed-off-by: Dongjoon Hyun <[email protected]>
1 parent c095b95 commit 4cc8101

File tree

6 files changed

+49
-2
lines changed

6 files changed

+49
-2
lines changed

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

Lines changed: 16 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -404,6 +404,14 @@ private[spark] object Config extends Logging {
404404
.toSequence
405405
.createWithDefault(Nil)
406406

407+
val KUBERNETES_DRIVER_POD_EXCLUDED_FEATURE_STEPS =
408+
ConfigBuilder("spark.kubernetes.driver.pod.excludedFeatureSteps")
409+
.doc("Class names to exclude from driver pod feature steps. Comma separated.")
410+
.version("4.1.0")
411+
.stringConf
412+
.toSequence
413+
.createWithDefault(Nil)
414+
407415
val KUBERNETES_EXECUTOR_POD_FEATURE_STEPS =
408416
ConfigBuilder("spark.kubernetes.executor.pod.featureSteps")
409417
.doc("Class name of an extra executor pod feature step implementing " +
@@ -416,6 +424,14 @@ private[spark] object Config extends Logging {
416424
.toSequence
417425
.createWithDefault(Nil)
418426

427+
val KUBERNETES_EXECUTOR_POD_EXCLUDED_FEATURE_STEPS =
428+
ConfigBuilder("spark.kubernetes.executor.pod.excludedFeatureSteps")
429+
.doc("Class name to exclude from executor pod feature steps. Comma separated.")
430+
.version("4.1.0")
431+
.stringConf
432+
.toSequence
433+
.createWithDefault(Nil)
434+
419435
val KUBERNETES_EXECUTOR_DECOMMISSION_LABEL =
420436
ConfigBuilder("spark.kubernetes.executor.decommissionLabel")
421437
.doc("Label to apply to a pod which is being decommissioned." +

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

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -72,7 +72,7 @@ class KubernetesDriverBuilder {
7272
}
7373
}
7474

75-
val features = Seq(
75+
val allFeatures = Seq(
7676
new BasicDriverFeatureStep(conf),
7777
new DriverKubernetesCredentialsFeatureStep(conf),
7878
new DriverServiceFeatureStep(conf),
@@ -85,6 +85,9 @@ class KubernetesDriverBuilder {
8585
new PodTemplateConfigMapStep(conf),
8686
new LocalDirsFeatureStep(conf)) ++ userFeatures
8787

88+
val features = allFeatures.filterNot(f =>
89+
conf.get(Config.KUBERNETES_DRIVER_POD_EXCLUDED_FEATURE_STEPS).contains(f.getClass.getName))
90+
8891
val spec = KubernetesDriverSpec(
8992
initialPod,
9093
driverPreKubernetesResources = Seq.empty,

resource-managers/kubernetes/core/src/main/scala/org/apache/spark/scheduler/cluster/k8s/KubernetesExecutorBuilder.scala

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -65,7 +65,7 @@ private[spark] class KubernetesExecutorBuilder {
6565
}
6666
}
6767

68-
val features = Seq(
68+
val allFeatures = Seq(
6969
new BasicExecutorFeatureStep(conf, secMgr, resourceProfile),
7070
new ExecutorKubernetesCredentialsFeatureStep(conf),
7171
new MountSecretsFeatureStep(conf),
@@ -74,6 +74,9 @@ private[spark] class KubernetesExecutorBuilder {
7474
new HadoopConfExecutorFeatureStep(conf),
7575
new LocalDirsFeatureStep(conf)) ++ userFeatures
7676

77+
val features = allFeatures.filterNot(f =>
78+
conf.get(Config.KUBERNETES_EXECUTOR_POD_EXCLUDED_FEATURE_STEPS).contains(f.getClass.getName))
79+
7780
val spec = KubernetesExecutorSpec(
7881
initialPod,
7982
executorKubernetesResources = Seq.empty)

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

Lines changed: 17 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -40,6 +40,8 @@ abstract class PodBuilderSuite extends SparkFunSuite {
4040

4141
protected def roleSpecificSchedulerNameConf: ConfigEntry[_]
4242

43+
protected def excludedFeatureStepsConf: ConfigEntry[_]
44+
4345
protected def userFeatureStepsConf: ConfigEntry[_]
4446

4547
protected def userFeatureStepWithExpectedAnnotation: (String, String)
@@ -91,6 +93,21 @@ abstract class PodBuilderSuite extends SparkFunSuite {
9193
assert(pod.container.getVolumeMounts.asScala.exists(_.getName == "so_long_two"))
9294
}
9395

96+
test("SPARK-52830: exclude a feature step") {
97+
val client = mockKubernetesClient()
98+
val sparkConf = baseConf.clone()
99+
.set(excludedFeatureStepsConf.key,
100+
"org.apache.spark.deploy.k8s.TestStepTwo")
101+
.set(userFeatureStepsConf.key,
102+
"org.apache.spark.deploy.k8s.TestStepTwo," +
103+
"org.apache.spark.deploy.k8s.TestStep")
104+
.set(templateFileConf.key, "template-file.yaml")
105+
val pod = buildPod(sparkConf, client)
106+
verifyPod(pod)
107+
assert(pod.container.getVolumeMounts.asScala.exists(_.getName == "so_long"))
108+
assert(!pod.container.getVolumeMounts.asScala.exists(_.getName == "so_long_two"))
109+
}
110+
94111
test("SPARK-37145: configure a custom test step with base config") {
95112
val client = mockKubernetesClient()
96113
val sparkConf = baseConf.clone()

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

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -38,6 +38,10 @@ class KubernetesDriverBuilderSuite extends PodBuilderSuite {
3838
Config.KUBERNETES_DRIVER_SCHEDULER_NAME
3939
}
4040

41+
override protected def excludedFeatureStepsConf: ConfigEntry[_] = {
42+
Config.KUBERNETES_DRIVER_POD_EXCLUDED_FEATURE_STEPS
43+
}
44+
4145
override protected def userFeatureStepsConf: ConfigEntry[_] = {
4246
Config.KUBERNETES_DRIVER_POD_FEATURE_STEPS
4347
}

resource-managers/kubernetes/core/src/test/scala/org/apache/spark/scheduler/cluster/k8s/KubernetesExecutorBuilderSuite.scala

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -42,6 +42,10 @@ class KubernetesExecutorBuilderSuite extends PodBuilderSuite {
4242
Config.KUBERNETES_EXECUTOR_SCHEDULER_NAME
4343
}
4444

45+
override protected def excludedFeatureStepsConf: ConfigEntry[_] = {
46+
Config.KUBERNETES_EXECUTOR_POD_EXCLUDED_FEATURE_STEPS
47+
}
48+
4549
override protected def userFeatureStepsConf: ConfigEntry[_] = {
4650
Config.KUBERNETES_EXECUTOR_POD_FEATURE_STEPS
4751
}

0 commit comments

Comments
 (0)