aboutsummaryrefslogtreecommitdiff
path: root/core/src/main/scala/org/apache/spark/util/Utils.scala
diff options
context:
space:
mode:
Diffstat (limited to 'core/src/main/scala/org/apache/spark/util/Utils.scala')
-rw-r--r--core/src/main/scala/org/apache/spark/util/Utils.scala23
1 files changed, 11 insertions, 12 deletions
diff --git a/core/src/main/scala/org/apache/spark/util/Utils.scala b/core/src/main/scala/org/apache/spark/util/Utils.scala
index e7160f164a..cdb322de3b 100644
--- a/core/src/main/scala/org/apache/spark/util/Utils.scala
+++ b/core/src/main/scala/org/apache/spark/util/Utils.scala
@@ -1689,17 +1689,15 @@ private[spark] object Utils extends Logging {
}
/**
- * Default maximum number of retries when binding to a port before giving up.
+ * Maximum number of retries when binding to a port before giving up.
*/
- val portMaxRetries: Int = {
- if (sys.props.contains("spark.testing")) {
+ def portMaxRetries(conf: SparkConf): Int = {
+ val maxRetries = conf.getOption("spark.port.maxRetries").map(_.toInt)
+ if (conf.contains("spark.testing")) {
// Set a higher number of retries for tests...
- sys.props.get("spark.port.maxRetries").map(_.toInt).getOrElse(100)
+ maxRetries.getOrElse(100)
} else {
- Option(SparkEnv.get)
- .flatMap(_.conf.getOption("spark.port.maxRetries"))
- .map(_.toInt)
- .getOrElse(16)
+ maxRetries.getOrElse(16)
}
}
@@ -1708,17 +1706,18 @@ private[spark] object Utils extends Logging {
* Each subsequent attempt uses 1 + the port used in the previous attempt (unless the port is 0).
*
* @param startPort The initial port to start the service on.
- * @param maxRetries Maximum number of retries to attempt.
- * A value of 3 means attempting ports n, n+1, n+2, and n+3, for example.
* @param startService Function to start service on a given port.
* This is expected to throw java.net.BindException on port collision.
+ * @param conf A SparkConf used to get the maximum number of retries when binding to a port.
+ * @param serviceName Name of the service.
*/
def startServiceOnPort[T](
startPort: Int,
startService: Int => (T, Int),
- serviceName: String = "",
- maxRetries: Int = portMaxRetries): (T, Int) = {
+ conf: SparkConf,
+ serviceName: String = ""): (T, Int) = {
val serviceString = if (serviceName.isEmpty) "" else s" '$serviceName'"
+ val maxRetries = portMaxRetries(conf)
for (offset <- 0 to maxRetries) {
// Do not increment port if startPort is 0, which is treated as a special port
val tryPort = if (startPort == 0) {