Skip to content

Commit 203605d

Browse files
committed
Merge branch 'master' into SPARK-6939
2 parents 74307cf + 38d4e9e commit 203605d

File tree

125 files changed

+3705
-1028
lines changed

Some content is hidden

Large Commits have some content hidden by default. Use the searchbox below for content that may be hidden.

125 files changed

+3705
-1028
lines changed

.rat-excludes

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -15,6 +15,7 @@ TAGS
1515
RELEASE
1616
control
1717
docs
18+
docker.properties.template
1819
fairscheduler.xml.template
1920
spark-defaults.conf.template
2021
log4j.properties

bin/spark-shell2.cmd

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -19,7 +19,7 @@ rem
1919

2020
set SPARK_HOME=%~dp0..
2121

22-
echo "%*" | findstr " --help -h" >nul
22+
echo "%*" | findstr " \<--help\> \<-h\>" >nul
2323
if %ERRORLEVEL% equ 0 (
2424
call :usage
2525
exit /b 0

conf/docker.properties.template

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,3 @@
1+
spark.mesos.executor.docker.image: <image built from `../docker/spark-mesos/Dockerfile`>
2+
spark.mesos.executor.docker.volumes: /usr/local/lib:/host/usr/local/lib:ro
3+
spark.mesos.executor.home: /opt/spark

core/src/main/scala/org/apache/spark/ExecutorAllocationManager.scala

Lines changed: 49 additions & 52 deletions
Original file line numberDiff line numberDiff line change
@@ -27,13 +27,20 @@ import org.apache.spark.util.{ThreadUtils, Clock, SystemClock, Utils}
2727
/**
2828
* An agent that dynamically allocates and removes executors based on the workload.
2929
*
30-
* The add policy depends on whether there are backlogged tasks waiting to be scheduled. If
31-
* the scheduler queue is not drained in N seconds, then new executors are added. If the queue
32-
* persists for another M seconds, then more executors are added and so on. The number added
33-
* in each round increases exponentially from the previous round until an upper bound on the
34-
* number of executors has been reached. The upper bound is based both on a configured property
35-
* and on the number of tasks pending: the policy will never increase the number of executor
36-
* requests past the number needed to handle all pending tasks.
30+
* The ExecutorAllocationManager maintains a moving target number of executors which is periodically
31+
* synced to the cluster manager. The target starts at a configured initial value and changes with
32+
* the number of pending and running tasks.
33+
*
34+
* Decreasing the target number of executors happens when the current target is more than needed to
35+
* handle the current load. The target number of executors is always truncated to the number of
36+
* executors that could run all current running and pending tasks at once.
37+
*
38+
* Increasing the target number of executors happens in response to backlogged tasks waiting to be
39+
* scheduled. If the scheduler queue is not drained in N seconds, then new executors are added. If
40+
* the queue persists for another M seconds, then more executors are added and so on. The number
41+
* added in each round increases exponentially from the previous round until an upper bound has been
42+
* reached. The upper bound is based both on a configured property and on the current number of
43+
* running and pending tasks, as described above.
3744
*
3845
* The rationale for the exponential increase is twofold: (1) Executors should be added slowly
3946
* in the beginning in case the number of extra executors needed turns out to be small. Otherwise,
@@ -105,8 +112,10 @@ private[spark] class ExecutorAllocationManager(
105112
// Number of executors to add in the next round
106113
private var numExecutorsToAdd = 1
107114

108-
// Number of executors that have been requested but have not registered yet
109-
private var numExecutorsPending = 0
115+
// The desired number of executors at this moment in time. If all our executors were to die, this
116+
// is the number of executors we would immediately want from the cluster manager.
117+
private var numExecutorsTarget =
118+
conf.getInt("spark.dynamicAllocation.initialExecutors", minNumExecutors)
110119

111120
// Executors that have been requested to be removed but have not been killed yet
112121
private val executorsPendingToRemove = new mutable.HashSet[String]
@@ -199,13 +208,6 @@ private[spark] class ExecutorAllocationManager(
199208
executor.awaitTermination(10, TimeUnit.SECONDS)
200209
}
201210

202-
/**
203-
* The number of executors we would have if the cluster manager were to fulfill all our existing
204-
* requests.
205-
*/
206-
private def targetNumExecutors(): Int =
207-
numExecutorsPending + executorIds.size - executorsPendingToRemove.size
208-
209211
/**
210212
* The maximum number of executors we would need under the current load to satisfy all running
211213
* and pending tasks, rounded up.
@@ -227,7 +229,7 @@ private[spark] class ExecutorAllocationManager(
227229
private def schedule(): Unit = synchronized {
228230
val now = clock.getTimeMillis
229231

230-
addOrCancelExecutorRequests(now)
232+
updateAndSyncNumExecutorsTarget(now)
231233

232234
removeTimes.retain { case (executorId, expireTime) =>
233235
val expired = now >= expireTime
@@ -239,26 +241,28 @@ private[spark] class ExecutorAllocationManager(
239241
}
240242

241243
/**
244+
* Updates our target number of executors and syncs the result with the cluster manager.
245+
*
242246
* Check to see whether our existing allocation and the requests we've made previously exceed our
243-
* current needs. If so, let the cluster manager know so that it can cancel pending requests that
244-
* are unneeded.
247+
* current needs. If so, truncate our target and let the cluster manager know so that it can
248+
* cancel pending requests that are unneeded.
245249
*
246250
* If not, and the add time has expired, see if we can request new executors and refresh the add
247251
* time.
248252
*
249253
* @return the delta in the target number of executors.
250254
*/
251-
private def addOrCancelExecutorRequests(now: Long): Int = synchronized {
252-
val currentTarget = targetNumExecutors
255+
private def updateAndSyncNumExecutorsTarget(now: Long): Int = synchronized {
253256
val maxNeeded = maxNumExecutorsNeeded
254257

255-
if (maxNeeded < currentTarget) {
258+
if (maxNeeded < numExecutorsTarget) {
256259
// The target number exceeds the number we actually need, so stop adding new
257-
// executors and inform the cluster manager to cancel the extra pending requests.
258-
val newTotalExecutors = math.max(maxNeeded, minNumExecutors)
259-
client.requestTotalExecutors(newTotalExecutors)
260+
// executors and inform the cluster manager to cancel the extra pending requests
261+
val oldNumExecutorsTarget = numExecutorsTarget
262+
numExecutorsTarget = math.max(maxNeeded, minNumExecutors)
263+
client.requestTotalExecutors(numExecutorsTarget)
260264
numExecutorsToAdd = 1
261-
updateNumExecutorsPending(newTotalExecutors)
265+
numExecutorsTarget - oldNumExecutorsTarget
262266
} else if (addTime != NOT_SET && now >= addTime) {
263267
val delta = addExecutors(maxNeeded)
264268
logDebug(s"Starting timer to add more executors (to " +
@@ -281,21 +285,30 @@ private[spark] class ExecutorAllocationManager(
281285
*/
282286
private def addExecutors(maxNumExecutorsNeeded: Int): Int = {
283287
// Do not request more executors if it would put our target over the upper bound
284-
val currentTarget = targetNumExecutors
285-
if (currentTarget >= maxNumExecutors) {
286-
logDebug(s"Not adding executors because there are already ${executorIds.size} " +
287-
s"registered and $numExecutorsPending pending executor(s) (limit $maxNumExecutors)")
288+
if (numExecutorsTarget >= maxNumExecutors) {
289+
val numExecutorsPending = numExecutorsTarget - executorIds.size
290+
logDebug(s"Not adding executors because there are already ${executorIds.size} registered " +
291+
s"and ${numExecutorsPending} pending executor(s) (limit $maxNumExecutors)")
288292
numExecutorsToAdd = 1
289293
return 0
290294
}
291295

292-
val actualMaxNumExecutors = math.min(maxNumExecutors, maxNumExecutorsNeeded)
293-
val newTotalExecutors = math.min(currentTarget + numExecutorsToAdd, actualMaxNumExecutors)
294-
val addRequestAcknowledged = testing || client.requestTotalExecutors(newTotalExecutors)
296+
val oldNumExecutorsTarget = numExecutorsTarget
297+
// There's no point in wasting time ramping up to the number of executors we already have, so
298+
// make sure our target is at least as much as our current allocation:
299+
numExecutorsTarget = math.max(numExecutorsTarget, executorIds.size)
300+
// Boost our target with the number to add for this round:
301+
numExecutorsTarget += numExecutorsToAdd
302+
// Ensure that our target doesn't exceed what we need at the present moment:
303+
numExecutorsTarget = math.min(numExecutorsTarget, maxNumExecutorsNeeded)
304+
// Ensure that our target fits within configured bounds:
305+
numExecutorsTarget = math.max(math.min(numExecutorsTarget, maxNumExecutors), minNumExecutors)
306+
307+
val addRequestAcknowledged = testing || client.requestTotalExecutors(numExecutorsTarget)
295308
if (addRequestAcknowledged) {
296-
val delta = updateNumExecutorsPending(newTotalExecutors)
309+
val delta = numExecutorsTarget - oldNumExecutorsTarget
297310
logInfo(s"Requesting $delta new executor(s) because tasks are backlogged" +
298-
s" (new desired total will be $newTotalExecutors)")
311+
s" (new desired total will be $numExecutorsTarget)")
299312
numExecutorsToAdd = if (delta == numExecutorsToAdd) {
300313
numExecutorsToAdd * 2
301314
} else {
@@ -304,23 +317,11 @@ private[spark] class ExecutorAllocationManager(
304317
delta
305318
} else {
306319
logWarning(
307-
s"Unable to reach the cluster manager to request $newTotalExecutors total executors!")
320+
s"Unable to reach the cluster manager to request $numExecutorsTarget total executors!")
308321
0
309322
}
310323
}
311324

312-
/**
313-
* Given the new target number of executors, update the number of pending executor requests,
314-
* and return the delta from the old number of pending requests.
315-
*/
316-
private def updateNumExecutorsPending(newTotalExecutors: Int): Int = {
317-
val newNumExecutorsPending =
318-
newTotalExecutors - executorIds.size + executorsPendingToRemove.size
319-
val delta = newNumExecutorsPending - numExecutorsPending
320-
numExecutorsPending = newNumExecutorsPending
321-
delta
322-
}
323-
324325
/**
325326
* Request the cluster manager to remove the given executor.
326327
* Return whether the request is received.
@@ -372,10 +373,6 @@ private[spark] class ExecutorAllocationManager(
372373
// as idle again so as not to forget that it is a candidate for removal. (see SPARK-4951)
373374
executorIds.filter(listener.isExecutorIdle).foreach(onExecutorIdle)
374375
logInfo(s"New executor $executorId has registered (new total is ${executorIds.size})")
375-
if (numExecutorsPending > 0) {
376-
numExecutorsPending -= 1
377-
logDebug(s"Decremented number of pending executors ($numExecutorsPending left)")
378-
}
379376
} else {
380377
logWarning(s"Duplicate executor $executorId has registered")
381378
}

core/src/main/scala/org/apache/spark/SecurityManager.scala

Lines changed: 15 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -150,8 +150,13 @@ import org.apache.spark.util.Utils
150150
* authorization. If not filter is in place the user is generally null and no authorization
151151
* can take place.
152152
*
153-
* Connection encryption (SSL) configuration is organized hierarchically. The user can configure
154-
* the default SSL settings which will be used for all the supported communication protocols unless
153+
* When authentication is being used, encryption can also be enabled by setting the option
154+
* spark.authenticate.enableSaslEncryption to true. This is only supported by communication
155+
* channels that use the network-common library, and can be used as an alternative to SSL in those
156+
* cases.
157+
*
158+
* SSL can be used for encryption for certain communication channels. The user can configure the
159+
* default SSL settings which will be used for all the supported communication protocols unless
155160
* they are overwritten by protocol specific settings. This way the user can easily provide the
156161
* common settings for all the protocols without disabling the ability to configure each one
157162
* individually.
@@ -412,6 +417,14 @@ private[spark] class SecurityManager(sparkConf: SparkConf)
412417
*/
413418
def isAuthenticationEnabled(): Boolean = authOn
414419

420+
/**
421+
* Checks whether SASL encryption should be enabled.
422+
* @return Whether to enable SASL encryption when connecting to services that support it.
423+
*/
424+
def isSaslEncryptionEnabled(): Boolean = {
425+
sparkConf.getBoolean("spark.authenticate.enableSaslEncryption", false)
426+
}
427+
415428
/**
416429
* Gets the user used for authenticating HTTP connections.
417430
* For now use a single hardcoded user.

core/src/main/scala/org/apache/spark/SparkContext.scala

Lines changed: 13 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -347,6 +347,19 @@ class SparkContext(config: SparkConf) extends Logging with ExecutorAllocationCli
347347
value
348348
}
349349

350+
/** Control our logLevel. This overrides any user-defined log settings.
351+
* @param logLevel The desired log level as a string.
352+
* Valid log levels include: ALL, DEBUG, ERROR, FATAL, INFO, OFF, TRACE, WARN
353+
*/
354+
def setLogLevel(logLevel: String) {
355+
val validLevels = Seq("ALL", "DEBUG", "ERROR", "FATAL", "INFO", "OFF", "TRACE", "WARN")
356+
if (!validLevels.contains(logLevel)) {
357+
throw new IllegalArgumentException(
358+
s"Supplied level $logLevel did not match one of: ${validLevels.mkString(",")}")
359+
}
360+
Utils.setLogLevel(org.apache.log4j.Level.toLevel(logLevel))
361+
}
362+
350363
try {
351364
_conf = config.clone()
352365
_conf.validateSettings()

core/src/main/scala/org/apache/spark/TestUtils.scala

Lines changed: 9 additions & 18 deletions
Original file line numberDiff line numberDiff line change
@@ -105,18 +105,23 @@ private[spark] object TestUtils {
105105
URI.create(s"string:///${name.replace(".", "/")}${SOURCE.extension}")
106106
}
107107

108-
private[spark] class JavaSourceFromString(val name: String, val code: String)
108+
private class JavaSourceFromString(val name: String, val code: String)
109109
extends SimpleJavaFileObject(createURI(name), SOURCE) {
110110
override def getCharContent(ignoreEncodingErrors: Boolean): String = code
111111
}
112112

113-
/** Creates a compiled class with the source file. Class file will be placed in destDir. */
113+
/** Creates a compiled class with the given name. Class file will be placed in destDir. */
114114
def createCompiledClass(
115115
className: String,
116116
destDir: File,
117-
sourceFile: JavaSourceFromString,
118-
classpathUrls: Seq[URL]): File = {
117+
toStringValue: String = "",
118+
baseClass: String = null,
119+
classpathUrls: Seq[URL] = Seq()): File = {
119120
val compiler = ToolProvider.getSystemJavaCompiler
121+
val extendsText = Option(baseClass).map { c => s" extends ${c}" }.getOrElse("")
122+
val sourceFile = new JavaSourceFromString(className,
123+
"public class " + className + extendsText + " implements java.io.Serializable {" +
124+
" @Override public String toString() { return \"" + toStringValue + "\"; }}")
120125

121126
// Calling this outputs a class file in pwd. It's easier to just rename the file than
122127
// build a custom FileManager that controls the output location.
@@ -139,18 +144,4 @@ private[spark] object TestUtils {
139144
assert(out.exists(), "Destination file not moved: " + out.getAbsolutePath())
140145
out
141146
}
142-
143-
/** Creates a compiled class with the given name. Class file will be placed in destDir. */
144-
def createCompiledClass(
145-
className: String,
146-
destDir: File,
147-
toStringValue: String = "",
148-
baseClass: String = null,
149-
classpathUrls: Seq[URL] = Seq()): File = {
150-
val extendsText = Option(baseClass).map { c => s" extends ${c}" }.getOrElse("")
151-
val sourceFile = new JavaSourceFromString(className,
152-
"public class " + className + extendsText + " implements java.io.Serializable {" +
153-
" @Override public String toString() { return \"" + toStringValue + "\"; }}")
154-
createCompiledClass(className, destDir, sourceFile, classpathUrls)
155-
}
156147
}

core/src/main/scala/org/apache/spark/api/java/JavaSparkContext.scala

Lines changed: 8 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -755,6 +755,14 @@ class JavaSparkContext(val sc: SparkContext)
755755
*/
756756
def getLocalProperty(key: String): String = sc.getLocalProperty(key)
757757

758+
/** Control our logLevel. This overrides any user-defined log settings.
759+
* @param logLevel The desired log level as a string.
760+
* Valid log levels include: ALL, DEBUG, ERROR, FATAL, INFO, OFF, TRACE, WARN
761+
*/
762+
def setLogLevel(logLevel: String) {
763+
sc.setLogLevel(logLevel)
764+
}
765+
758766
/**
759767
* Assigns a group ID to all the jobs started by this thread until the group ID is set to a
760768
* different value or cleared.

0 commit comments

Comments
 (0)