Skip to content

Commit 036550d

Browse files
committed
SPARK-1713. [YARN] Use a threadpool for launching executor containers
1 parent 07ee4a2 commit 036550d

File tree

2 files changed

+19
-2
lines changed

2 files changed

+19
-2
lines changed

docs/running-on-yarn.md

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -125,6 +125,13 @@ Most of the configs are the same for Spark on YARN as for other deployment modes
125125
the environment of the executor launcher.
126126
</td>
127127
</tr>
128+
<tr>
129+
<td><code>spark.yarn.containerLauncherMaxThreads</code></td>
130+
<td>25</td>
131+
<td>
132+
The maximum number of threads to use in the application master for launching executor containers.
133+
</td>
134+
</tr>
128135
</table>
129136

130137
# Launching Spark on YARN

yarn/common/src/main/scala/org/apache/spark/deploy/yarn/YarnAllocator.scala

Lines changed: 12 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -18,7 +18,7 @@
1818
package org.apache.spark.deploy.yarn
1919

2020
import java.util.{List => JList}
21-
import java.util.concurrent.ConcurrentHashMap
21+
import java.util.concurrent._
2222
import java.util.concurrent.atomic.AtomicInteger
2323

2424
import scala.collection.JavaConversions._
@@ -32,6 +32,8 @@ import org.apache.spark.{Logging, SecurityManager, SparkConf, SparkEnv}
3232
import org.apache.spark.scheduler.{SplitInfo, TaskSchedulerImpl}
3333
import org.apache.spark.scheduler.cluster.CoarseGrainedSchedulerBackend
3434

35+
import com.google.common.util.concurrent.ThreadFactoryBuilder
36+
3537
object AllocationType extends Enumeration {
3638
type AllocationType = Value
3739
val HOST, RACK, ANY = Value
@@ -95,6 +97,14 @@ private[yarn] abstract class YarnAllocator(
9597
protected val (preferredHostToCount, preferredRackToCount) =
9698
generateNodeToWeight(conf, preferredNodes)
9799

100+
private val launcherPool = new ThreadPoolExecutor(
101+
// max pool size of Integer.MAX_VALUE is ignored because we use an unbounded queue
102+
sparkConf.getInt("spark.yarn.containerLauncherMaxThreads", 25), Integer.MAX_VALUE,
103+
1, TimeUnit.MINUTES,
104+
new LinkedBlockingQueue[Runnable](),
105+
new ThreadFactoryBuilder().setNameFormat("ContainerLauncher #%d").setDaemon(true).build())
106+
launcherPool.allowCoreThreadTimeOut(true)
107+
98108
def getNumExecutorsRunning: Int = numExecutorsRunning.intValue
99109

100110
def getNumExecutorsFailed: Int = numExecutorsFailed.intValue
@@ -283,7 +293,7 @@ private[yarn] abstract class YarnAllocator(
283293
executorMemory,
284294
executorCores,
285295
securityMgr)
286-
new Thread(executorRunnable).start()
296+
launcherPool.execute(executorRunnable)
287297
}
288298
}
289299
logDebug("""

0 commit comments

Comments
 (0)