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

Commit 018f4d8

Browse files
committed
Address comments.
1 parent 2b5dcac commit 018f4d8

File tree

3 files changed

+82
-106
lines changed

3 files changed

+82
-106
lines changed

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

Lines changed: 0 additions & 24 deletions
Original file line numberDiff line numberDiff line change
@@ -103,35 +103,11 @@ package object config extends Logging {
103103
.longConf
104104
.createWithDefault(1)
105105

106-
private[spark] val INIT_CONTAINER_JARS_DOWNLOAD_LOCATION =
107-
ConfigBuilder("spark.kubernetes.mountdependencies.jarsDownloadDir")
108-
.doc("Location to download jars to in the driver and executors. When using" +
109-
" spark-submit, this directory must be empty and will be mounted as an empty directory" +
110-
" volume on the driver and executor pod.")
111-
.stringConf
112-
.createWithDefault("/var/spark-data/spark-jars")
113-
114106
private[spark] val KUBERNETES_EXECUTOR_LIMIT_CORES =
115107
ConfigBuilder("spark.kubernetes.executor.limit.cores")
116108
.doc("Specify the hard cpu limit for a single executor pod")
117109
.stringConf
118110
.createOptional
119111

120112
private[spark] val KUBERNETES_NODE_SELECTOR_PREFIX = "spark.kubernetes.node.selector."
121-
122-
private[spark] def resolveK8sMaster(rawMasterString: String): String = {
123-
if (!rawMasterString.startsWith("k8s://")) {
124-
throw new IllegalArgumentException("Master URL should start with k8s:// in Kubernetes mode.")
125-
}
126-
val masterWithoutK8sPrefix = rawMasterString.replaceFirst("k8s://", "")
127-
if (masterWithoutK8sPrefix.startsWith("http://")
128-
|| masterWithoutK8sPrefix.startsWith("https://")) {
129-
masterWithoutK8sPrefix
130-
} else {
131-
val resolvedURL = s"https://$masterWithoutK8sPrefix"
132-
logInfo("No scheme specified for kubernetes master URL, so defaulting to https. Resolved" +
133-
s" URL is $resolvedURL")
134-
resolvedURL
135-
}
136-
}
137113
}

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

Lines changed: 4 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -47,7 +47,6 @@ private[spark] class ExecutorPodFactoryImpl(sparkConf: SparkConf)
4747

4848
private val executorExtraClasspath = sparkConf.get(
4949
org.apache.spark.internal.config.EXECUTOR_CLASS_PATH)
50-
private val executorJarsDownloadDir = sparkConf.get(INIT_CONTAINER_JARS_DOWNLOAD_LOCATION)
5150

5251
private val executorLabels = ConfigurationUtils.parsePrefixedKeyValuePairs(
5352
sparkConf,
@@ -94,7 +93,7 @@ private[spark] class ExecutorPodFactoryImpl(sparkConf: SparkConf)
9493
MEMORY_OVERHEAD_MIN_MIB))
9594
private val executorMemoryWithOverhead = executorMemoryMiB + memoryOverheadMiB
9695

97-
private val executorCores = sparkConf.getDouble("spark.executor.cores", 1d)
96+
private val executorCores = sparkConf.getDouble("spark.executor.cores", 1)
9897
private val executorLimitCores = sparkConf.getOption(KUBERNETES_EXECUTOR_LIMIT_CORES.key)
9998

10099
override def createExecutorPod(
@@ -108,7 +107,7 @@ private[spark] class ExecutorPodFactoryImpl(sparkConf: SparkConf)
108107

109108
// hostname must be no longer than 63 characters, so take the last 63 characters of the pod
110109
// name as the hostname. This preserves uniqueness since the end of name contains
111-
// executorId and applicationId
110+
// executorId
112111
val hostname = name.substring(Math.max(0, name.length - 63))
113112
val resolvedExecutorLabels = Map(
114113
SPARK_EXECUTOR_ID_LABEL -> executorId,
@@ -139,15 +138,14 @@ private[spark] class ExecutorPodFactoryImpl(sparkConf: SparkConf)
139138
new EnvVarBuilder().withName(s"$ENV_JAVA_OPT_PREFIX$index").withValue(opt).build()
140139
}
141140
}.getOrElse(Seq.empty[EnvVar])
142-
val executorEnv = (Seq(
141+
val executorEnv = Seq(
143142
(ENV_EXECUTOR_PORT, executorPort.toString),
144143
(ENV_DRIVER_URL, driverUrl),
145144
// Executor backend expects integral value for executor cores, so round it up to an int.
146145
(ENV_EXECUTOR_CORES, math.ceil(executorCores).toInt.toString),
147146
(ENV_EXECUTOR_MEMORY, executorMemoryString),
148147
(ENV_APPLICATION_ID, applicationId),
149-
(ENV_EXECUTOR_ID, executorId),
150-
(ENV_MOUNTED_CLASSPATH, s"$executorJarsDownloadDir/*")) ++ executorEnvs)
148+
(ENV_EXECUTOR_ID, executorId))
151149
.map(env => new EnvVarBuilder()
152150
.withName(env._1)
153151
.withValue(env._2)

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

Lines changed: 78 additions & 76 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,7 @@ import java.io.Closeable
2020
import java.net.InetAddress
2121
import java.util.concurrent.{ConcurrentHashMap, ExecutorService, ScheduledExecutorService, TimeUnit}
2222
import java.util.concurrent.atomic.{AtomicInteger, AtomicLong, AtomicReference}
23+
import javax.annotation.concurrent.GuardedBy
2324

2425
import io.fabric8.kubernetes.api.model._
2526
import io.fabric8.kubernetes.client.{KubernetesClient, KubernetesClientException, Watcher}
@@ -49,9 +50,11 @@ private[spark] class KubernetesClusterSchedulerBackend(
4950

5051
private val EXECUTOR_ID_COUNTER = new AtomicLong(0L)
5152
private val RUNNING_EXECUTOR_PODS_LOCK = new Object
52-
// Indexed by executor IDs and guarded by RUNNING_EXECUTOR_PODS_LOCK.
53+
// Indexed by executor IDs
54+
@GuardedBy("RUNNING_EXECUTOR_PODS_LOCK")
5355
private val runningExecutorsToPods = new mutable.HashMap[String, Pod]
54-
// Indexed by executor pod names and guarded by RUNNING_EXECUTOR_PODS_LOCK.
56+
// Indexed by executor pod names
57+
@GuardedBy("RUNNING_EXECUTOR_PODS_LOCK")
5558
private val runningPodsToExecutors = new mutable.HashMap[String, String]
5659
private val executorPodsByIPs = new ConcurrentHashMap[String, Pod]()
5760
private val podsWithKnownExitReasons = new ConcurrentHashMap[String, ExecutorExited]()
@@ -105,21 +108,44 @@ private[spark] class KubernetesClusterSchedulerBackend(
105108

106109
override def run(): Unit = {
107110
handleDisconnectedExecutors()
111+
val executorsToAllocate = mutable.Map[String, Pod]()
112+
val currentTotalRegisteredExecutors = totalRegisteredExecutors.get
113+
val currentTotalExpectedExecutors = totalExpectedExecutors.get
114+
val currentNodeToLocalTaskCount = getNodesWithLocalTaskCounts
115+
if (currentTotalRegisteredExecutors < runningExecutorsToPods.size) {
116+
logDebug("Waiting for pending executors before scaling")
117+
} else if (currentTotalExpectedExecutors <= runningExecutorsToPods.size) {
118+
logDebug("Maximum allowed executor limit reached. Not scaling up further.")
119+
} else {
120+
val nodeToLocalTaskCount = getNodesWithLocalTaskCounts
121+
for (i <- 0 until math.min(
122+
currentTotalExpectedExecutors - runningExecutorsToPods.size, podAllocationSize)) {
123+
val executorId = EXECUTOR_ID_COUNTER.incrementAndGet().toString
124+
val executorPod = executorPodFactory.createExecutorPod(
125+
executorId,
126+
applicationId(),
127+
driverUrl,
128+
conf.getExecutorEnv,
129+
driverPod,
130+
nodeToLocalTaskCount)
131+
executorsToAllocate(executorId) = executorPod
132+
logInfo(
133+
s"Requesting a new executor, total executors is now ${runningExecutorsToPods.size}")
134+
}
135+
}
136+
val allocatedExecutors = executorsToAllocate.mapValues { pod =>
137+
Utils.tryLog {
138+
kubernetesClient.pods().create(pod)
139+
}
140+
}
108141
RUNNING_EXECUTOR_PODS_LOCK.synchronized {
109-
if (totalRegisteredExecutors.get() < runningExecutorsToPods.size) {
110-
logDebug("Waiting for pending executors before scaling")
111-
} else if (totalExpectedExecutors.get() <= runningExecutorsToPods.size) {
112-
logDebug("Maximum allowed executor limit reached. Not scaling up further.")
113-
} else {
114-
val nodeToLocalTaskCount = getNodesWithLocalTaskCounts
115-
for (i <- 0 until math.min(
116-
totalExpectedExecutors.get - runningExecutorsToPods.size, podAllocationSize)) {
117-
val (executorId, pod) = allocateNewExecutorPod(nodeToLocalTaskCount)
118-
runningExecutorsToPods.put(executorId, pod)
119-
runningPodsToExecutors.put(pod.getMetadata.getName, executorId)
120-
logInfo(
121-
s"Requesting a new executor, total executors is now ${runningExecutorsToPods.size}")
122-
}
142+
allocatedExecutors.map {
143+
case (executorId, attemptedAllocatedExecutor) =>
144+
attemptedAllocatedExecutor.map { successfullyAllocatedExecutor =>
145+
runningExecutorsToPods.put(executorId, successfullyAllocatedExecutor)
146+
runningPodsToExecutors.put(
147+
successfullyAllocatedExecutor.getMetadata.getName, executorId)
148+
}
123149
}
124150
}
125151
}
@@ -128,25 +154,25 @@ private[spark] class KubernetesClusterSchedulerBackend(
128154
// For each disconnected executor, synchronize with the loss reasons that may have been found
129155
// by the executor pod watcher. If the loss reason was discovered by the watcher,
130156
// inform the parent class with removeExecutor.
131-
disconnectedPodsByExecutorIdPendingRemoval.keys().asScala.foreach { case (executorId) =>
132-
val executorPod = disconnectedPodsByExecutorIdPendingRemoval.get(executorId)
133-
val knownExitReason = Option(podsWithKnownExitReasons.remove(
134-
executorPod.getMetadata.getName))
135-
knownExitReason.fold {
136-
removeExecutorOrIncrementLossReasonCheckCount(executorId)
137-
} { executorExited =>
138-
logWarning(s"Removing executor $executorId with loss reason " + executorExited.message)
139-
removeExecutor(executorId, executorExited)
140-
// We keep around executors that have exit conditions caused by the application. This
141-
// allows them to be debugged later on. Otherwise, mark them as to be deleted from the
142-
// the API server.
143-
if (!executorExited.exitCausedByApp) {
144-
logInfo(s"Executor $executorId failed because of a framework error.")
145-
deleteExecutorFromClusterAndDataStructures(executorId)
146-
} else {
147-
logInfo(s"Executor $executorId exited because of the application.")
157+
disconnectedPodsByExecutorIdPendingRemoval.asScala.foreach {
158+
case (executorId, executorPod) =>
159+
val knownExitReason = Option(podsWithKnownExitReasons.remove(
160+
executorPod.getMetadata.getName))
161+
knownExitReason.fold {
162+
removeExecutorOrIncrementLossReasonCheckCount(executorId)
163+
} { executorExited =>
164+
logWarning(s"Removing executor $executorId with loss reason " + executorExited.message)
165+
removeExecutor(executorId, executorExited)
166+
// We keep around executors that have exit conditions caused by the application. This
167+
// allows them to be debugged later on. Otherwise, mark them as to be deleted from the
168+
// the API server.
169+
if (!executorExited.exitCausedByApp) {
170+
logInfo(s"Executor $executorId failed because of a framework error.")
171+
deleteExecutorFromClusterAndDataStructures(executorId)
172+
} else {
173+
logInfo(s"Executor $executorId exited because of the application.")
174+
}
148175
}
149-
}
150176
}
151177
}
152178

@@ -163,12 +189,17 @@ private[spark] class KubernetesClusterSchedulerBackend(
163189
def deleteExecutorFromClusterAndDataStructures(executorId: String): Unit = {
164190
disconnectedPodsByExecutorIdPendingRemoval.remove(executorId)
165191
executorReasonCheckAttemptCounts -= executorId
166-
RUNNING_EXECUTOR_PODS_LOCK.synchronized {
192+
podsWithKnownExitReasons -= executorId
193+
val maybeExecutorPodToDelete = RUNNING_EXECUTOR_PODS_LOCK.synchronized {
167194
runningExecutorsToPods.remove(executorId).map { pod =>
168-
kubernetesClient.pods().delete(pod)
169195
runningPodsToExecutors.remove(pod.getMetadata.getName)
170-
}.getOrElse(logWarning(s"Unable to remove pod for unknown executor $executorId"))
196+
pod
197+
}.orElse {
198+
logWarning(s"Unable to remove pod for unknown executor $executorId")
199+
None
200+
}
171201
}
202+
maybeExecutorPodToDelete.foreach(pod => kubernetesClient.pods().delete(pod))
172203
}
173204
}
174205

@@ -203,25 +234,23 @@ private[spark] class KubernetesClusterSchedulerBackend(
203234
// TODO investigate why Utils.tryLogNonFatalError() doesn't work in this context.
204235
// When using Utils.tryLogNonFatalError some of the code fails but without any logs or
205236
// indication as to why.
206-
try {
207-
RUNNING_EXECUTOR_PODS_LOCK.synchronized {
208-
runningExecutorsToPods.values.foreach(kubernetesClient.pods().delete(_))
237+
Utils.tryLogNonFatalError {
238+
val executorPodsToDelete = RUNNING_EXECUTOR_PODS_LOCK.synchronized {
239+
val runningExecutorPodsCopy = Seq(runningExecutorsToPods.values.toSeq: _*)
209240
runningExecutorsToPods.clear()
210241
runningPodsToExecutors.clear()
242+
runningExecutorPodsCopy
211243
}
244+
kubernetesClient.pods().delete(executorPodsToDelete: _*)
212245
executorPodsByIPs.clear()
213246
val resource = executorWatchResource.getAndSet(null)
214247
if (resource != null) {
215248
resource.close()
216249
}
217-
} catch {
218-
case e: Throwable => logError("Uncaught exception while shutting down controllers.", e)
219250
}
220-
try {
251+
Utils.tryLogNonFatalError {
221252
logInfo("Closing kubernetes client")
222253
kubernetesClient.close()
223-
} catch {
224-
case e: Throwable => logError("Uncaught exception closing Kubernetes client.", e)
225254
}
226255
}
227256

@@ -231,7 +260,7 @@ private[spark] class KubernetesClusterSchedulerBackend(
231260
*/
232261
private def getNodesWithLocalTaskCounts() : Map[String, Int] = {
233262
val nodeToLocalTaskCount = mutable.Map[String, Int]() ++
234-
KubernetesClusterSchedulerBackend.this.synchronized {
263+
synchronized {
235264
hostToLocalTaskCount
236265
}
237266
for (pod <- executorPodsByIPs.values().asScala) {
@@ -247,58 +276,31 @@ private[spark] class KubernetesClusterSchedulerBackend(
247276
nodeToLocalTaskCount.toMap[String, Int]
248277
}
249278

250-
/**
251-
* Allocates a new executor pod
252-
*
253-
* @param nodeToLocalTaskCount A map of K8s cluster nodes to the number of tasks that could
254-
* benefit from data locality if an executor launches on the cluster
255-
* node.
256-
* @return A tuple of the new executor name and the Pod data structure.
257-
*/
258-
private def allocateNewExecutorPod(nodeToLocalTaskCount: Map[String, Int]): (String, Pod) = {
259-
val executorId = EXECUTOR_ID_COUNTER.incrementAndGet().toString
260-
val executorPod = executorPodFactory.createExecutorPod(
261-
executorId,
262-
applicationId(),
263-
driverUrl,
264-
conf.getExecutorEnv,
265-
driverPod,
266-
nodeToLocalTaskCount)
267-
try {
268-
(executorId, kubernetesClient.pods.create(executorPod))
269-
} catch {
270-
case throwable: Throwable =>
271-
logError("Failed to allocate executor pod.", throwable)
272-
throw throwable
273-
}
274-
}
275-
276279
override def doRequestTotalExecutors(requestedTotal: Int): Future[Boolean] = Future[Boolean] {
277280
totalExpectedExecutors.set(requestedTotal)
278281
true
279282
}
280283

281284
override def doKillExecutors(executorIds: Seq[String]): Future[Boolean] = Future[Boolean] {
285+
val podsToDelete = mutable.Buffer[Pod]()
282286
RUNNING_EXECUTOR_PODS_LOCK.synchronized {
283287
for (executor <- executorIds) {
284288
val maybeRemovedExecutor = runningExecutorsToPods.remove(executor)
285289
maybeRemovedExecutor.foreach { executorPod =>
286-
kubernetesClient.pods().delete(executorPod)
287290
disconnectedPodsByExecutorIdPendingRemoval.put(executor, executorPod)
288291
runningPodsToExecutors.remove(executorPod.getMetadata.getName)
292+
podsToDelete += executorPod
289293
}
290294
if (maybeRemovedExecutor.isEmpty) {
291295
logWarning(s"Unable to remove pod for unknown executor $executor")
292296
}
293297
}
294298
}
299+
kubernetesClient.pods().delete(podsToDelete: _*)
295300
true
296301
}
297302

298303
def getExecutorPodByIP(podIP: String): Option[Pod] = {
299-
// Note: Per https://github.com/databricks/scala-style-guide#concurrency, we don't
300-
// want to be switching to scala.collection.concurrent.Map on
301-
// executorPodsByIPs.
302304
val pod = executorPodsByIPs.get(podIP)
303305
Option(pod)
304306
}

0 commit comments

Comments
 (0)