Skip to content
Closed
Show file tree
Hide file tree
Changes from 5 commits
Commits
Show all changes
115 commits
Select commit Hold shift + click to select a range
4b398d0
expose UI data as json in new endpoints
squito Feb 6, 2015
d1a8c92
add sbt-revolved plugin, to make it easier to start & stop http serve…
squito Feb 6, 2015
b252e7a
small cleanup of accidental changes
squito Feb 6, 2015
e0356b6
put new test expectation files in rat excludes (is this OK?)
squito Feb 6, 2015
bceb3a9
set http response code on error, some testing
squito Feb 6, 2015
f0264a7
switch to using jersey for metrics json
squito Feb 9, 2015
cba1ef6
add security (maybe?) for metrics json
squito Feb 11, 2015
4fdc39f
refactor case insensitive enum parsing
squito Feb 17, 2015
a4ab5aa
add in explicit dependency on jersey 1.9 -- maven wasn't happy before
squito Feb 19, 2015
cd37845
switch to using java.util.Dates for times
squito Feb 23, 2015
b6a96a8
compare json by AST, not string
squito Feb 23, 2015
0c12b50
Merge branch 'master' into SPARK-3454_w_jersey
squito Mar 10, 2015
c7d884f
fix merge conflicts
squito Mar 10, 2015
0dc3ea7
if app isnt found, reload apps from FS before giving up
squito Mar 10, 2015
1b78cb7
fix some import ordering
squito Mar 10, 2015
017c755
add in metrics now available
squito Mar 10, 2015
56edce0
style
squito Mar 10, 2015
5598f19
delete some unnecessary code, more to go
squito Mar 11, 2015
18a8c45
Merge branch 'master' into SPARK-3454_w_jersey
squito Mar 16, 2015
b86e2b0
style
squito Mar 16, 2015
654cecf
move all the status api POJOs to one file
squito Mar 16, 2015
d05f7a9
dont use case classes for status api POJOs, since they have binary co…
squito Mar 16, 2015
3377e61
scaladoc
squito Mar 16, 2015
00e9cc5
more style
squito Mar 16, 2015
9c0c125
add accumulableInfo
squito Mar 16, 2015
f7df095
add test for accumulables, and discover that I need update after all
squito Mar 16, 2015
56d2fc7
jersey needs asm for SPARK_PREPEND_CLASSES to work
squito Mar 16, 2015
e91750a
Merge branch 'master' into SPARK-3454_w_jersey
squito Mar 17, 2015
b4c75ed
fix merge conflicts; need to widen visibility in a few cases
squito Mar 17, 2015
36e4062
SparkUI needs to know about startTime, so it can list its own applica…
squito Mar 18, 2015
84cd497
AllJobsPage should still report correct completed & failed job count,…
squito Mar 19, 2015
190c17a
StagePage should distinguish no task data, from unknown stage
squito Mar 19, 2015
dddbd29
stages have attempt; jobs are sorted; resource for all attempts for o…
squito Mar 19, 2015
0c96147
better error msgs for bad stageId vs bad attemptId
squito Mar 19, 2015
97d411f
json endpoint for one job
squito Mar 19, 2015
73f1378
test json; also add test cases for cleaned stages & jobs
squito Mar 19, 2015
7fd156a
refactor jsonDiff to avoid code duplication
squito Mar 19, 2015
5f9df24
style
squito Mar 20, 2015
b86bcb0
update test to look at one stage attempt
squito Mar 20, 2015
dbfc7bf
style
squito Mar 20, 2015
fef6605
some utils for working w/ new "enum" format
squito Mar 20, 2015
2382bef
switch to using new "enum"
squito Mar 20, 2015
0be5120
Merge branch 'master' into SPARK-3454_w_jersey
squito Mar 23, 2015
1f361c8
update rat-excludes
squito Mar 23, 2015
a066055
set visibility on a lot of classes
squito Mar 23, 2015
a61a43c
oops, remove accidental checkin
squito Mar 23, 2015
99764e1
Merge branch 'SPARK-3454_w_jersey' into SPARK-3454
squito Mar 23, 2015
9f28b7e
ack, more cleanup
squito Mar 23, 2015
51eaedb
more visibility fixes
squito Mar 23, 2015
25cd894
if only given day, assume GMT
squito Mar 23, 2015
eaf3bbb
style
squito Mar 25, 2015
e48ba32
rename
squito Mar 25, 2015
a4b1397
stage metric distributions
squito Mar 31, 2015
aaba896
wire up task summary
squito Mar 31, 2015
b2efcaf
cleanup, combine stage-related paths into one resource
squito Mar 31, 2015
ad27de8
error handling on quantile values
squito Apr 1, 2015
63eb4a6
tests for taskSummary
squito Apr 1, 2015
fdfc181
stage/taskList
squito Apr 1, 2015
db61211
get JobProgressListener directly from UI
squito Apr 1, 2015
f5a5196
undo removal of renderJson from MasterPage, since there is no substit…
squito Apr 1, 2015
9922be0
Merge branch 'master' into stage_distributions
squito Apr 1, 2015
674f8dc
more explicit about total numbers of jobs & stages vs. number retained
squito Apr 1, 2015
1578a4a
doc
squito Apr 1, 2015
2c8b7ee
rats
squito Apr 1, 2015
b4a7863
fix compile error
squito Apr 1, 2015
1f53a66
style
squito Apr 1, 2015
e031719
fixes from review
squito Apr 8, 2015
b136e39
Revert "add sbt-revolved plugin, to make it easier to start & stop ht…
squito Apr 8, 2015
d2ef58d
revert changes that had HistoryServer refresh the application listing…
squito Apr 8, 2015
101a698
style
squito Apr 8, 2015
f05ae89
add in ExecutorSummaryInfo for MiMa :(
squito Apr 13, 2015
d493b38
Merge branch 'master' into SPARK-3454
squito Apr 13, 2015
cf86175
style
squito Apr 13, 2015
9ea682c
go back to good ol' java enums
squito Apr 15, 2015
c22b11f
fix compile error
squito Apr 15, 2015
f2e63ad
Merge branch 'master' into SPARK-3454
squito Apr 23, 2015
4c92af6
style
squito Apr 23, 2015
1ed0d07
Merge branch 'master' into SPARK-3454
squito Apr 27, 2015
0c6f968
update deps
squito Apr 27, 2015
a9c5cf1
undo changes superceeded by master
squito Apr 27, 2015
a325563
style
squito Apr 27, 2015
7bd4d15
delete security test, since it doesnt do anything
squito Apr 27, 2015
a157a2f
style
squito Apr 27, 2015
4a234d3
avoid jersey-media-json-jackson b/c of potential version conflicts
squito Apr 27, 2015
d2bde77
update error handling & scoping
squito Apr 27, 2015
39ac29c
move EnumUtil
squito Apr 27, 2015
ba3d9d2
upper case enums
squito Apr 27, 2015
2e19be2
lazily convert ApplicationInfo to avoid memory overhead
squito Apr 27, 2015
b2f8b91
@DeveloperApi
squito Apr 27, 2015
31c79ce
asm no longer needed for SPARK_PREPEND_CLASSES
squito Apr 27, 2015
52bbae8
StorageListener & StorageStatusListener needs to synchronize internal…
squito Apr 27, 2015
f48a7b0
docs
squito Apr 28, 2015
9d889d6
undo some unnecessary changes
squito Apr 28, 2015
7bf1811
move MetricHelper so mima doesnt think its exposed; comments
squito Apr 28, 2015
acb7ef6
fix indentation
squito Apr 28, 2015
dc8a7fe
style, fix errant comments
squito Apr 28, 2015
f90680e
Merge branch 'master' into SPARK-3454
squito Apr 28, 2015
14ac3ed
jersey-core needs to be explicit; move version & scope to parent pom.xml
squito Apr 28, 2015
befff0c
review feedback
squito Apr 28, 2015
2af11e5
Merge branch 'master' into SPARK-3454
squito May 1, 2015
188762c
multi-attempt
squito May 1, 2015
b87cd63
add sbt-revolved plugin, to make it easier to start & stop http serve…
squito Feb 6, 2015
c9bae1c
handle multiple attempts per app
squito May 1, 2015
9e51400
style
squito May 1, 2015
67008b4
rats
squito May 1, 2015
7f3bc4e
Revert "add sbt-revolved plugin, to make it easier to start & stop ht…
squito May 1, 2015
56db31e
update tests for mulit-attempt
squito May 1, 2015
cbaf287
Merge branch 'master' into SPARK-3454
squito May 1, 2015
cc1febf
add docs on the metrics-as-json api
squito May 5, 2015
ec140a2
create @Private
squito May 5, 2015
3347b72
mark EnumUtil as @Private
squito May 5, 2015
f016182
change all constructors json-pojo class constructors to be private[sp…
squito May 5, 2015
5ae02ad
Merge branch 'master' into SPARK-3454
squito May 5, 2015
5e78b4f
fix rendering problems
squito May 5, 2015
da1e35f
typos etc.
squito May 5, 2015
File filter

Filter by extension

Filter by extension


Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
3 changes: 3 additions & 0 deletions .rat-excludes
Original file line number Diff line number Diff line change
Expand Up @@ -65,3 +65,6 @@ logs
.*scalastyle-output.xml
.*dependency-reduced-pom.xml
known_translations
json_expectation
local-1422981759269/*
local-1422981780767/*
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

is this OK? or should the apache header go in the test resource files as well?

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

the header shouldn't go into those files

5 changes: 5 additions & 0 deletions core/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -214,6 +214,11 @@
<artifactId>json4s-jackson_${scala.binary.version}</artifactId>
<version>3.2.10</version>
</dependency>
<dependency>
<groupId>com.fasterxml.jackson.module</groupId>
<artifactId>jackson-module-scala_2.10</artifactId>
<version>2.3.1</version>
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

this seems like a common dependency by our users. how well does jackson work against different versions?

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

this is rxin: ah i c we are already adding it. the question stands for @pwendell / @andrewor14 - should we shade this? Is it robust enough to not shade?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

not exactly what you're asking but this module doesn't always play nicely with different versions of jackson itself. So if somehow jackson gets bumped up to a later version b/c of some transitive dependency, often this needs to get bumped up to go along with it.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

v2.4.4 is already included in master so if you rebase you won't have to include it again

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

what i was asking @pwendell/@andrewor14 was the probability this breaks user applications (or user applications break Spark) due to version conflicts for both Jackson & the Scala module.

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@squito just a reminder that you should update this version to ${fasterxml.jackson.version}.

</dependency>
<dependency>
<groupId>org.apache.mesos</groupId>
<artifactId>mesos</artifactId>
Expand Down
23 changes: 23 additions & 0 deletions core/src/main/java/org/apache/spark/status/api/StageStatus.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,23 @@
package org.apache.spark.status.api;/*
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

move this below the license?

* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/

public enum StageStatus {
Active,
Complete,
Failed,
Pending
}
Original file line number Diff line number Diff line change
Expand Up @@ -98,4 +98,5 @@ private[spark] object JsonProtocol {
("executors" -> obj.executors.toList.map(writeExecutorRunner)) ~
("finishedexecutors" -> obj.finishedExecutors.toList.map(writeExecutorRunner))
}

}
Original file line number Diff line number Diff line change
@@ -0,0 +1,56 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.spark.deploy.history

import javax.servlet.http.HttpServletRequest

import org.apache.spark.status.{UIRoot, StatusJsonRoute}
import org.apache.spark.status.api.ApplicationInfo
import org.apache.spark.deploy.master.{ApplicationInfo => InternalApplicationInfo}

class AllApplicationsJsonRoute(val uiRoot: UIRoot) extends StatusJsonRoute[Seq[ApplicationInfo]] {

override def renderJson(request: HttpServletRequest): Seq[ApplicationInfo] = {
//TODO filter on some query params, eg. completed, minStartTime, etc
uiRoot.getApplicationInfoList
}

}

object AllApplicationsJsonRoute {
def appHistoryInfoToPublicAppInfo(app: ApplicationHistoryInfo): ApplicationInfo = {
ApplicationInfo(
id = app.id,
name = app.name,
startTime = app.startTime,
endTime = app.endTime,
sparkUser = app.sparkUser,
completed = app.completed
)
}

def convertApplicationInfo(internal: InternalApplicationInfo, completed: Boolean): ApplicationInfo = {
ApplicationInfo(
id = internal.id,
name = internal.desc.name,
startTime = internal.startTime,
endTime = internal.endTime,
sparkUser = internal.desc.user,
completed = completed
)
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -35,7 +35,7 @@ private[spark] abstract class ApplicationHistoryProvider {
*
* @return List of all know applications.
*/
def getListing(): Iterable[ApplicationHistoryInfo]
def getListing(refresh: Boolean): Iterable[ApplicationHistoryInfo]

/**
* Returns the Spark UI for a specific application.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -122,7 +122,10 @@ private[history] class FsHistoryProvider(conf: SparkConf) extends ApplicationHis
}
}

override def getListing() = applications.values
override def getListing(refresh: Boolean) = {
if (refresh) checkForLogs()
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

nit:

if (blah) {
  code
}

We only use the shorthand when using if..else as a ternary.

applications.values
}

override def getAppUI(appId: String): Option[SparkUI] = {
try {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -34,7 +34,7 @@ private[spark] class HistoryPage(parent: HistoryServer) extends WebUIPage("") {
val requestedIncomplete =
Option(request.getParameter("showIncomplete")).getOrElse("false").toBoolean

val allApps = parent.getApplicationList().filter(_.completed != requestedIncomplete)
val allApps = parent.getApplicationList(true).filter(_.completed != requestedIncomplete)
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Same comment. Forcing a refresh of the listing is expensive, and doesn't sound related to this PR.

val actualFirst = if (requestedFirst < allApps.size) requestedFirst else 0
val apps = allApps.slice(actualFirst, Math.min(actualFirst + pageSize, allApps.size))

Expand Down Expand Up @@ -67,7 +67,7 @@ private[spark] class HistoryPage(parent: HistoryServer) extends WebUIPage("") {

<h4>
Showing {actualFirst + 1}-{last + 1} of {allApps.size}
{if (requestedIncomplete) "(Incomplete applications)"}
({if (requestedIncomplete) "Incomplete" else "Complete"} applications)
<span style="float: right">
{
if (actualPage > 1) {
Expand All @@ -90,7 +90,7 @@ private[spark] class HistoryPage(parent: HistoryServer) extends WebUIPage("") {
</h4> ++
appTable
} else {
<h4>No completed applications found!</h4> ++
<h4>No {if (requestedIncomplete) "running" else "completed"} applications found!</h4> ++
<p>Did you specify the correct logging directory?
Please verify your setting of <span style="font-style:italic">
spark.history.fs.logDirectory</span> and whether you have the permissions to
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,9 @@ import java.util.NoSuchElementException
import javax.servlet.http.{HttpServlet, HttpServletRequest, HttpServletResponse}

import com.google.common.cache._
import org.apache.spark.deploy.master.ui.MasterApplicationJsonRoute
import org.apache.spark.status.api.ApplicationInfo
import org.apache.spark.status.{UIRoot, JsonRequestHandler}
import org.eclipse.jetty.servlet.{ServletContextHandler, ServletHolder}

import org.apache.spark.{Logging, SecurityManager, SparkConf}
Expand All @@ -45,7 +48,7 @@ class HistoryServer(
provider: ApplicationHistoryProvider,
securityManager: SecurityManager,
port: Int)
extends WebUI(securityManager, port, conf) with Logging {
extends WebUI(securityManager, port, conf) with Logging with UIRoot {

// How many applications to retain
private val retainedApplications = conf.getInt("spark.history.retainedApplications", 50)
Expand All @@ -71,6 +74,7 @@ class HistoryServer(
protected override def doGet(req: HttpServletRequest, res: HttpServletResponse): Unit = {
val parts = Option(req.getPathInfo()).getOrElse("").split("/")
if (parts.length < 2) {
logError("bad path info!")
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

log the path too?

res.sendError(HttpServletResponse.SC_BAD_REQUEST,
s"Unexpected path info in request (URI = ${req.getRequestURI()}")
return
Expand Down Expand Up @@ -98,6 +102,10 @@ class HistoryServer(
}
}

def getSparkUI(appKey: String): Option[SparkUI] = {
Option(appCache.get(appKey))
}

initialize()

/**
Expand All @@ -108,6 +116,11 @@ class HistoryServer(
*/
def initialize() {
attachPage(new HistoryPage(this))

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

nit: delete

val jsonHandler = new JsonRequestHandler(this, securityManager)
attachHandler(jsonHandler.jsonContextHandler)


attachHandler(createStaticHandler(SparkUI.STATIC_RESOURCE_DIR, "/static"))

val contextHandler = new ServletContextHandler
Expand Down Expand Up @@ -145,7 +158,11 @@ class HistoryServer(
*
* @return List of all known applications.
*/
def getApplicationList() = provider.getListing()
def getApplicationList(refresh: Boolean) = provider.getListing(refresh)

def getApplicationInfoList: Seq[ApplicationInfo] = {
getApplicationList(true).map{AllApplicationsJsonRoute.appHistoryInfoToPublicAppInfo}.toSeq
}

/**
* Returns the provider configuration to show in the listing page.
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,34 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.spark.deploy.history

import javax.servlet.http.HttpServletRequest

import org.apache.spark.status.{UIRoot, JsonRequestHandler, StatusJsonRoute}
import org.apache.spark.status.api.ApplicationInfo

class OneApplicationJsonRoute(val uiRoot: UIRoot) extends StatusJsonRoute[ApplicationInfo] {
override def renderJson(request: HttpServletRequest): ApplicationInfo = {
val appIdOpt = JsonRequestHandler.extractAppId(request.getPathInfo)
appIdOpt.map{ appId =>
val apps = uiRoot.getApplicationInfoList.find{_.id == appId}
apps.getOrElse(throw new IllegalArgumentException("unknown app: " + appId))
}.getOrElse{
throw new IllegalArgumentException("no application id specified")
}
}
}
14 changes: 7 additions & 7 deletions core/src/main/scala/org/apache/spark/deploy/master/Master.scala
Original file line number Diff line number Diff line change
Expand Up @@ -715,9 +715,9 @@ private[spark] class Master(

/**
* Rebuild a new SparkUI from the given application's event logs.
* Return whether this is successful.
* Return the UI if successful, else None
*/
def rebuildSparkUI(app: ApplicationInfo): Boolean = {
def rebuildSparkUI(app: ApplicationInfo): Option[SparkUI] = {
val appName = app.desc.name
val notFoundBasePath = HistoryServer.UI_PATH_PREFIX + "/not-found"
try {
Expand All @@ -726,7 +726,7 @@ private[spark] class Master(
.getOrElse {
// Event logging is not enabled for this application
app.desc.appUiUrl = notFoundBasePath
return false
return None
}

val fs = Utils.getHadoopFileSystem(eventLogFile, hadoopConf)
Expand All @@ -738,7 +738,7 @@ private[spark] class Master(
logWarning(msg)
msg = URLEncoder.encode(msg, "UTF-8")
app.desc.appUiUrl = notFoundBasePath + s"?msg=$msg&title=$title"
return false
return None
}

val (logInput, sparkVersion) = EventLoggingListener.openEventLog(new Path(eventLogFile), fs)
Expand All @@ -754,7 +754,7 @@ private[spark] class Master(
webUi.attachSparkUI(ui)
// Application UI is successfully rebuilt, so link the Master UI to it
app.desc.appUiUrl = ui.basePath
true
Some(ui)
} catch {
case fnf: FileNotFoundException =>
// Event logging is enabled for this application, but no event logs are found
Expand All @@ -764,7 +764,7 @@ private[spark] class Master(
msg += " Did you specify the correct logging directory?"
msg = URLEncoder.encode(msg, "UTF-8")
app.desc.appUiUrl = notFoundBasePath + s"?msg=$msg&title=$title"
false
None
case e: Exception =>
// Relay exception message to application UI page
val title = s"Application history load error (${app.id})"
Expand All @@ -773,7 +773,7 @@ private[spark] class Master(
logError(msg, e)
msg = URLEncoder.encode(msg, "UTF-8")
app.desc.appUiUrl = notFoundBasePath + s"?msg=$msg&exception=$exception&title=$title"
false
None
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -23,9 +23,8 @@ import scala.concurrent.Await
import scala.xml.Node

import akka.pattern.ask
import org.json4s.JValue

import org.apache.spark.deploy.{ExecutorState, JsonProtocol}
import org.apache.spark.deploy.ExecutorState
import org.apache.spark.deploy.DeployMessages.{MasterStateResponse, RequestMasterState}
import org.apache.spark.deploy.master.ExecutorDesc
import org.apache.spark.ui.{UIUtils, WebUIPage}
Expand All @@ -36,17 +35,6 @@ private[spark] class ApplicationPage(parent: MasterWebUI) extends WebUIPage("app
private val master = parent.masterActorRef
private val timeout = parent.timeout

/** Executor details for a particular application */
override def renderJson(request: HttpServletRequest): JValue = {
val appId = request.getParameter("appId")
val stateFuture = (master ? RequestMasterState)(timeout).mapTo[MasterStateResponse]
val state = Await.result(stateFuture, timeout)
val app = state.activeApps.find(_.id == appId).getOrElse({
state.completedApps.find(_.id == appId).getOrElse(null)
})
JsonProtocol.writeApplicationInfo(app)
}

/** Executor details for a particular application */
def render(request: HttpServletRequest): Seq[Node] = {
val appId = request.getParameter("appId")
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,44 @@
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.spark.deploy.master.ui

import javax.servlet.http.HttpServletRequest

import akka.pattern.ask

import org.apache.spark.deploy.DeployMessages.{RequestMasterState, MasterStateResponse}
import org.apache.spark.status.StatusJsonRoute
import org.apache.spark.status.api.ApplicationInfo

import scala.concurrent.Await

class MasterApplicationJsonRoute(val parent: MasterWebUI) extends StatusJsonRoute[ApplicationInfo] {
private val master = parent.masterActorRef
private val timeout = parent.timeout


override def renderJson(request: HttpServletRequest): ApplicationInfo = {
//TODO not really the app id
val appId = request.getPathInfo()
println("pathInfo = " + request.getPathInfo())
val stateFuture = (master ? RequestMasterState)(timeout).mapTo[MasterStateResponse]
val state = Await.result(stateFuture, timeout)
state.activeApps.find(_.id == appId).orElse({
state.completedApps.find(_.id == appId)
}).map{MasterJsonRoute.masterAppInfoToPublicAppInfo}.getOrElse(null)
}
}
Loading