Skip to content

Commit b6dba10

Browse files
CodingCatrxin
authored andcommitted
Merge pull request #556 from CodingCat/JettyUtil. Closes #556.
[SPARK-1060] startJettyServer should explicitly use IP information https://spark-project.atlassian.net/browse/SPARK-1060 In the current implementation, the webserver in Master/Worker is started with val (srv, bPort) = JettyUtils.startJettyServer("0.0.0.0", port, handlers) inside startJettyServer: val server = new Server(currentPort) //here, the Server will take "0.0.0.0" as the hostname, i.e. will always bind to the IP address of the first NIC this can cause wrong IP binding, e.g. if the host has two NICs, N1 and N2, the user specify the SPARK_LOCAL_IP as the N2's IP address, however, when starting the web server, for the reason stated above, it will always bind to the N1's address Author: CodingCat <zhunansjtu@gmail.com> == Merge branch commits == commit 6c6d9a8ccc9ec4590678a3b34cb03df19092029d Author: CodingCat <zhunansjtu@gmail.com> Date: Thu Feb 6 14:53:34 2014 -0500 startJettyServer should explicitly use IP information
1 parent 2ef37c9 commit b6dba10

File tree

5 files changed

+12
-9
lines changed

5 files changed

+12
-9
lines changed

core/src/main/scala/org/apache/spark/deploy/master/ui/MasterWebUI.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -45,7 +45,7 @@ class MasterWebUI(val master: Master, requestedPort: Int) extends Logging {
4545

4646
def start() {
4747
try {
48-
val (srv, bPort) = JettyUtils.startJettyServer("0.0.0.0", port, handlers)
48+
val (srv, bPort) = JettyUtils.startJettyServer(host, port, handlers)
4949
server = Some(srv)
5050
boundPort = Some(bPort)
5151
logInfo("Started Master web UI at http://%s:%d".format(host, boundPort.get))

core/src/main/scala/org/apache/spark/deploy/worker/ui/WorkerWebUI.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -56,7 +56,7 @@ class WorkerWebUI(val worker: Worker, val workDir: File, requestedPort: Option[I
5656

5757
def start() {
5858
try {
59-
val (srv, bPort) = JettyUtils.startJettyServer("0.0.0.0", port, handlers)
59+
val (srv, bPort) = JettyUtils.startJettyServer(host, port, handlers)
6060
server = Some(srv)
6161
boundPort = Some(bPort)
6262
logInfo("Started Worker web UI at http://%s:%d".format(host, bPort))

core/src/main/scala/org/apache/spark/ui/JettyUtils.scala

Lines changed: 6 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -30,12 +30,14 @@ import org.eclipse.jetty.server.handler.{ResourceHandler, HandlerList, ContextHa
3030
import org.eclipse.jetty.util.thread.QueuedThreadPool
3131

3232
import org.apache.spark.Logging
33+
import java.net.InetSocketAddress
3334

3435

3536
/** Utilities for launching a web server using Jetty's HTTP Server class */
3637
private[spark] object JettyUtils extends Logging {
3738
// Base type for a function that returns something based on an HTTP request. Allows for
3839
// implicit conversion from many types of functions to jetty Handlers.
40+
3941
type Responder[T] = HttpServletRequest => T
4042

4143
// Conversions from various types of Responder's to jetty Handlers
@@ -92,12 +94,13 @@ private[spark] object JettyUtils extends Logging {
9294
}
9395

9496
/**
95-
* Attempts to start a Jetty server at the supplied ip:port which uses the supplied handlers.
97+
* Attempts to start a Jetty server at the supplied hostName:port which uses the supplied handlers.
9698
*
9799
* If the desired port number is contented, continues incrementing ports until a free port is
98100
* found. Returns the chosen port and the jetty Server object.
99101
*/
100-
def startJettyServer(ip: String, port: Int, handlers: Seq[(String, Handler)]): (Server, Int) = {
102+
def startJettyServer(hostName: String, port: Int, handlers: Seq[(String, Handler)]): (Server, Int) = {
103+
101104
val handlersToRegister = handlers.map { case(path, handler) =>
102105
val contextHandler = new ContextHandler(path)
103106
contextHandler.setHandler(handler)
@@ -109,7 +112,7 @@ private[spark] object JettyUtils extends Logging {
109112

110113
@tailrec
111114
def connect(currentPort: Int): (Server, Int) = {
112-
val server = new Server(currentPort)
115+
val server = new Server(new InetSocketAddress(hostName, currentPort))
113116
val pool = new QueuedThreadPool
114117
pool.setDaemon(true)
115118
server.setThreadPool(pool)

core/src/main/scala/org/apache/spark/ui/SparkUI.scala

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -54,7 +54,7 @@ private[spark] class SparkUI(sc: SparkContext) extends Logging {
5454
/** Bind the HTTP server which backs this web interface */
5555
def bind() {
5656
try {
57-
val (srv, usedPort) = JettyUtils.startJettyServer("0.0.0.0", port, allHandlers)
57+
val (srv, usedPort) = JettyUtils.startJettyServer(host, port, allHandlers)
5858
logInfo("Started Spark Web UI at http://%s:%d".format(host, usedPort))
5959
server = Some(srv)
6060
boundPort = Some(usedPort)

core/src/test/scala/org/apache/spark/ui/UISuite.scala

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -32,15 +32,15 @@ class UISuite extends FunSuite {
3232
case Failure(e) =>
3333
// Either case server port is busy hence setup for test complete
3434
}
35-
val (jettyServer1, boundPort1) = JettyUtils.startJettyServer("localhost", startPort, Seq())
36-
val (jettyServer2, boundPort2) = JettyUtils.startJettyServer("localhost", startPort, Seq())
35+
val (jettyServer1, boundPort1) = JettyUtils.startJettyServer("0.0.0.0", startPort, Seq())
36+
val (jettyServer2, boundPort2) = JettyUtils.startJettyServer("0.0.0.0", startPort, Seq())
3737
// Allow some wiggle room in case ports on the machine are under contention
3838
assert(boundPort1 > startPort && boundPort1 < startPort + 10)
3939
assert(boundPort2 > boundPort1 && boundPort2 < boundPort1 + 10)
4040
}
4141

4242
test("jetty binds to port 0 correctly") {
43-
val (jettyServer, boundPort) = JettyUtils.startJettyServer("localhost", 0, Seq())
43+
val (jettyServer, boundPort) = JettyUtils.startJettyServer("0.0.0.0", 0, Seq())
4444
assert(jettyServer.getState === "STARTED")
4545
assert(boundPort != 0)
4646
Try {new ServerSocket(boundPort)} match {

0 commit comments

Comments
 (0)