Skip to content

Commit 56cc7fb

Browse files
committed
First cut implementation of Streaming UI.
1 parent 6f986f0 commit 56cc7fb

File tree

3 files changed

+207
-0
lines changed

3 files changed

+207
-0
lines changed

streaming/src/main/scala/org/apache/spark/streaming/StreamingContext.scala

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -40,6 +40,7 @@ import org.apache.spark.streaming.dstream._
4040
import org.apache.spark.streaming.receivers._
4141
import org.apache.spark.streaming.scheduler._
4242
import org.apache.hadoop.conf.Configuration
43+
import org.apache.spark.streaming.ui.StreamingUI
4344

4445
/**
4546
* Main entry point for Spark Streaming functionality. It provides methods used to create
@@ -158,6 +159,9 @@ class StreamingContext private[streaming] (
158159

159160
private[streaming] val waiter = new ContextWaiter
160161

162+
private[streaming] val ui = new StreamingUI(this)
163+
ui.bind()
164+
161165
/**
162166
* Return the associated Spark context
163167
*/
Lines changed: 131 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,131 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
18+
package org.apache.spark.streaming.ui
19+
20+
import scala.collection.mutable.SynchronizedQueue
21+
import scala.xml.Node
22+
23+
import javax.servlet.http.HttpServletRequest
24+
import org.eclipse.jetty.servlet.ServletContextHandler
25+
26+
import org.apache.spark.Logging
27+
import org.apache.spark.streaming.StreamingContext
28+
import org.apache.spark.streaming.scheduler.{BatchInfo, StreamingListener, StreamingListenerBatchCompleted}
29+
import org.apache.spark.ui.{ServerInfo, SparkUI}
30+
import org.apache.spark.ui.JettyUtils._
31+
import org.apache.spark.util.{Distribution, Utils}
32+
33+
private[spark] class StreamingUIListener() extends StreamingListener {
34+
35+
private val batchInfos = new SynchronizedQueue[BatchInfo]
36+
private val maxBatchInfos = 100
37+
38+
override def onBatchCompleted(batchStarted: StreamingListenerBatchCompleted) {
39+
batchInfos.enqueue(batchStarted.batchInfo)
40+
if (batchInfos.size > maxBatchInfos) batchInfos.dequeue()
41+
}
42+
43+
def processingDelayDistribution = extractDistribution(_.processingDelay)
44+
45+
def schedulingDelayDistribution = extractDistribution(_.schedulingDelay)
46+
47+
def totalDelay = extractDistribution(_.totalDelay)
48+
49+
def extractDistribution(getMetric: BatchInfo => Option[Long]): Option[Distribution] = {
50+
Distribution(batchInfos.flatMap(getMetric(_)).map(_.toDouble))
51+
}
52+
53+
def numBatchInfos = batchInfos.size
54+
}
55+
56+
private[spark] class StreamingUI(ssc: StreamingContext) extends Logging {
57+
58+
private val sc = ssc.sparkContext
59+
private val conf = sc.conf
60+
private val appName = sc.appName
61+
private val bindHost = Utils.localHostName()
62+
private val publicHost = Option(System.getenv("SPARK_PUBLIC_DNS")).getOrElse(bindHost)
63+
private val port = conf.getInt("spark.streaming.ui.port", StreamingUI.DEFAULT_PORT)
64+
private val securityManager = sc.env.securityManager
65+
private val listener = new StreamingUIListener()
66+
private val handlers: Seq[ServletContextHandler] = {
67+
Seq(
68+
createServletHandler("/",
69+
(request: HttpServletRequest) => render(request), securityManager),
70+
createStaticHandler(SparkUI.STATIC_RESOURCE_DIR, "/static")
71+
)
72+
}
73+
74+
private var serverInfo: Option[ServerInfo] = None
75+
76+
ssc.addStreamingListener(listener)
77+
78+
def bind() {
79+
try {
80+
serverInfo = Some(startJettyServer(bindHost, port, handlers, sc.conf))
81+
logInfo("Started Spark Streaming Web UI at http://%s:%d".format(publicHost, boundPort))
82+
} catch {
83+
case e: Exception =>
84+
logError("Failed to create Spark JettyUtils", e)
85+
System.exit(1)
86+
}
87+
}
88+
89+
def boundPort: Int = serverInfo.map(_.boundPort).getOrElse(-1)
90+
91+
private def render(request: HttpServletRequest): Seq[Node] = {
92+
val batchStatsTable = generateBatchStatsTable()
93+
val content = batchStatsTable
94+
UIUtils.headerStreamingPage(content, "", appName, "Spark Streaming Overview")
95+
}
96+
97+
private def generateBatchStatsTable(): Seq[Node] = {
98+
def getQuantiles(timeDistributionOption: Option[Distribution]) = {
99+
timeDistributionOption.get.getQuantiles().map { ms => Utils.msDurationToString(ms.toLong) }
100+
}
101+
val numBatches = listener.numBatchInfos
102+
val table = if (numBatches > 0) {
103+
val processingDelayQuantilesRow =
104+
"Processing Times" +: getQuantiles(listener.processingDelayDistribution)
105+
val schedulingDelayQuantilesRow =
106+
"Scheduling Delay:" +: getQuantiles(listener.processingDelayDistribution)
107+
val totalDelayQuantilesRow =
108+
"End-to-end Delay:" +: getQuantiles(listener.totalDelay)
109+
110+
val headerRow = Seq("Metric", "Min", "25th percentile",
111+
"Median", "75th percentile", "Max")
112+
val dataRows: Seq[Seq[String]] = Seq(
113+
processingDelayQuantilesRow,
114+
schedulingDelayQuantilesRow,
115+
totalDelayQuantilesRow
116+
)
117+
Some(UIUtils.listingTable(headerRow, dataRows, fixedWidth = true))
118+
} else {
119+
None
120+
}
121+
122+
val content =
123+
<h4>Batch Processing Statistics</h4> ++
124+
<div>{table.getOrElse("No statistics have been generated yet.")}</div>
125+
content
126+
}
127+
}
128+
129+
object StreamingUI {
130+
val DEFAULT_PORT = 6060
131+
}
Lines changed: 72 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,72 @@
1+
package org.apache.spark.streaming.ui
2+
3+
import scala.xml.Node
4+
import org.apache.spark.ui.Page
5+
6+
private[spark] object UIUtils {
7+
8+
import org.apache.spark.ui.UIUtils.prependBaseUri
9+
10+
def headerStreamingPage(
11+
content: => Seq[Node],
12+
basePath: String,
13+
appName: String,
14+
title: String): Seq[Node] = {
15+
val overview = {
16+
<li><a href={prependBaseUri(basePath)}>Overview</a></li>
17+
}
18+
19+
<html>
20+
<head>
21+
<meta http-equiv="Content-type" content="text/html; charset=utf-8" />
22+
<link rel="stylesheet" href={prependBaseUri("/static/bootstrap.min.css")}
23+
type="text/css" />
24+
<link rel="stylesheet" href={prependBaseUri("/static/webui.css")}
25+
type="text/css" />
26+
<script src={prependBaseUri("/static/sorttable.js")} ></script>
27+
<title>{appName} - {title}</title>
28+
</head>
29+
<body>
30+
<div class="navbar navbar-static-top">
31+
<div class="navbar-inner">
32+
<a href={prependBaseUri(basePath, "/")} class="brand">
33+
<img src={prependBaseUri("/static/spark-logo-77x50px-hd.png")} />
34+
</a>
35+
<ul class="nav">
36+
{overview}
37+
</ul>
38+
<p class="navbar-text pull-right"><strong>{appName}</strong> application UI</p>
39+
</div>
40+
</div>
41+
42+
<div class="container-fluid">
43+
<div class="row-fluid">
44+
<div class="span12">
45+
<h3 style="vertical-align: bottom; display: inline-block;">
46+
{title}
47+
</h3>
48+
</div>
49+
</div>
50+
{content}
51+
</div>
52+
</body>
53+
</html>
54+
}
55+
56+
def listingTable[T](
57+
headers: Seq[String],
58+
makeRow: T => Seq[Node],
59+
rows: Seq[T],
60+
fixedWidth: Boolean = false): Seq[Node] = {
61+
org.apache.spark.ui.UIUtils.listingTable(headers, makeRow, rows, fixedWidth)
62+
}
63+
64+
def listingTable[T](
65+
headers: Seq[String],
66+
rows: Seq[Seq[String]],
67+
fixedWidth: Boolean = false
68+
): Seq[Node] = {
69+
def makeRow(data: Seq[String]): Seq[Node] = <tr> {data.map(d => <td>{d}</td>)} </tr>
70+
org.apache.spark.ui.UIUtils.listingTable(headers, makeRow, rows, fixedWidth)
71+
}
72+
}

0 commit comments

Comments
 (0)