- Show additional metrics
+ Show Additional Metrics
@@ -86,19 +77,19 @@ private[ui] class RDDPage(parent: StorageTab) extends WebUIPage("rdd") {
-
Data Distribution on {workers.size} Executors
+ Data Distribution on {rddStorageInfo.dataDistribution.size} Executors
{workerTable}
-
{blocks.size} Partitions
+ {rddStorageInfo.partitions.size} Partitions
{blockTable}
;
- UIUtils.headerSparkPage("RDD Storage Info for " + rddInfo.name, content, parent)
+ UIUtils.headerSparkPage("RDD Storage Info for " + rddStorageInfo.name, content, parent)
}
/** Header fields for the worker table */
@@ -116,34 +107,32 @@ private[ui] class RDDPage(parent: StorageTab) extends WebUIPage("rdd") {
"Executors")
/** Render an HTML row representing a worker */
- private def workerRow(worker: (Int, StorageStatus)): Seq[Node] = {
- val (rddId, status) = worker
+ private def workerRow(worker: RDDDataDistribution): Seq[Node] = {
- {status.blockManagerId.host + ":" + status.blockManagerId.port} |
+ {worker.address} |
- {Utils.bytesToString(status.memUsedByRdd(rddId))}
- ({Utils.bytesToString(status.memRemaining)} Remaining)
+ {Utils.bytesToString(worker.memoryUsed)}
+ ({Utils.bytesToString(worker.memoryRemaining)} Remaining)
|
- {Utils.bytesToString(status.diskUsedByRdd(rddId))} |
+ {Utils.bytesToString(worker.diskUsed)} |
}
/** Render an HTML row representing a block */
- private def blockRow(row: (BlockId, BlockStatus, Seq[String])): Seq[Node] = {
- val (id, block, locations) = row
+ private def blockRow(row: RDDPartitionInfo): Seq[Node] = {
- {id} |
+ {row.blockName} |
- {block.storageLevel.description}
+ {row.storageLevel}
|
-
- {Utils.bytesToString(block.memSize)}
+ |
+ {Utils.bytesToString(row.memoryUsed)}
|
-
- {Utils.bytesToString(block.diskSize)}
+ |
+ {Utils.bytesToString(row.diskUsed)}
|
- {locations.map(l => {l} )}
+ {row.executors.map(l => {l} )}
|
}
diff --git a/core/src/main/scala/org/apache/spark/ui/storage/StoragePage.scala b/core/src/main/scala/org/apache/spark/ui/storage/StoragePage.scala
index 59dc6b547c1d8..07db783c572cf 100644
--- a/core/src/main/scala/org/apache/spark/ui/storage/StoragePage.scala
+++ b/core/src/main/scala/org/apache/spark/ui/storage/StoragePage.scala
@@ -22,7 +22,7 @@ import javax.servlet.http.HttpServletRequest
import scala.xml.Node
import org.apache.spark.storage.RDDInfo
-import org.apache.spark.ui.{WebUIPage, UIUtils}
+import org.apache.spark.ui.{UIUtils, WebUIPage}
import org.apache.spark.util.Utils
/** Page showing list of RDD's currently stored in the cluster */
diff --git a/core/src/main/scala/org/apache/spark/ui/storage/StorageTab.scala b/core/src/main/scala/org/apache/spark/ui/storage/StorageTab.scala
index 045bd784990d1..0351749700962 100644
--- a/core/src/main/scala/org/apache/spark/ui/storage/StorageTab.scala
+++ b/core/src/main/scala/org/apache/spark/ui/storage/StorageTab.scala
@@ -35,6 +35,8 @@ private[ui] class StorageTab(parent: SparkUI) extends SparkUITab(parent, "storag
/**
* :: DeveloperApi ::
* A SparkListener that prepares information to be displayed on the BlockManagerUI.
+ *
+ * This class is thread-safe (unlike JobProgressListener)
*/
@DeveloperApi
class StorageListener(storageStatusListener: StorageStatusListener) extends SparkListener {
@@ -43,7 +45,9 @@ class StorageListener(storageStatusListener: StorageStatusListener) extends Spar
def storageStatusList: Seq[StorageStatus] = storageStatusListener.storageStatusList
/** Filter RDD info to include only those with cached partitions */
- def rddInfoList: Seq[RDDInfo] = _rddInfoMap.values.filter(_.numCachedPartitions > 0).toSeq
+ def rddInfoList: Seq[RDDInfo] = synchronized {
+ _rddInfoMap.values.filter(_.numCachedPartitions > 0).toSeq
+ }
/** Update the storage info of the RDDs whose blocks are among the given updated blocks */
private def updateRDDInfo(updatedBlocks: Seq[(BlockId, BlockStatus)]): Unit = {
diff --git a/core/src/test/resources/HistoryServerExpectations/application_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/application_list_json_expectation.json
new file mode 100644
index 0000000000000..ce4fe80b66aa5
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/application_list_json_expectation.json
@@ -0,0 +1,53 @@
+[ {
+ "id" : "local-1430917381534",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-05-06T13:03:00.893GMT",
+ "endTime" : "2015-05-06T13:03:11.398GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1426533911241",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "attemptId" : "2",
+ "startTime" : "2015-03-17T23:11:50.242GMT",
+ "endTime" : "2015-03-17T23:12:25.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ }, {
+ "attemptId" : "1",
+ "startTime" : "2015-03-16T19:25:10.242GMT",
+ "endTime" : "2015-03-16T19:25:45.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1425081759269",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-28T00:02:38.277GMT",
+ "endTime" : "2015-02-28T00:02:46.912GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1422981780767",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:59.720GMT",
+ "endTime" : "2015-02-03T16:43:08.731GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1422981759269",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:38.277GMT",
+ "endTime" : "2015-02-03T16:42:46.912GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/complete_stage_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/complete_stage_list_json_expectation.json
new file mode 100644
index 0000000000000..31ac9beea8788
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/complete_stage_list_json_expectation.json
@@ -0,0 +1,67 @@
+[ {
+ "status" : "COMPLETE",
+ "stageId" : 3,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 162,
+ "inputBytes" : 160,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 0,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "count at
:17",
+ "details" : "org.apache.spark.rdd.RDD.count(RDD.scala:910)\n$line19.$read$$iwC$$iwC$$iwC$$iwC.(:17)\n$line19.$read$$iwC$$iwC$$iwC.(:22)\n$line19.$read$$iwC$$iwC.(:24)\n$line19.$read$$iwC.(:26)\n$line19.$read.(:28)\n$line19.$read$.(:32)\n$line19.$read$.()\n$line19.$eval$.(:7)\n$line19.$eval$.()\n$line19.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+}, {
+ "status" : "COMPLETE",
+ "stageId" : 1,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 3476,
+ "inputBytes" : 28000128,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 13180,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "map at :14",
+ "details" : "org.apache.spark.rdd.RDD.map(RDD.scala:271)\n$line10.$read$$iwC$$iwC$$iwC$$iwC.(:14)\n$line10.$read$$iwC$$iwC$$iwC.(:19)\n$line10.$read$$iwC$$iwC.(:21)\n$line10.$read$$iwC.(:23)\n$line10.$read.(:25)\n$line10.$read$.(:29)\n$line10.$read$.()\n$line10.$eval$.(:7)\n$line10.$eval$.()\n$line10.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+}, {
+ "status" : "COMPLETE",
+ "stageId" : 0,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 4338,
+ "inputBytes" : 0,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 0,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "count at :15",
+ "details" : "org.apache.spark.rdd.RDD.count(RDD.scala:910)\n$line9.$read$$iwC$$iwC$$iwC$$iwC.(:15)\n$line9.$read$$iwC$$iwC$$iwC.(:20)\n$line9.$read$$iwC$$iwC.(:22)\n$line9.$read$$iwC.(:24)\n$line9.$read.(:26)\n$line9.$read$.(:30)\n$line9.$read$.()\n$line9.$eval$.(:7)\n$line9.$eval$.()\n$line9.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/completed_app_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/completed_app_list_json_expectation.json
new file mode 100644
index 0000000000000..ce4fe80b66aa5
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/completed_app_list_json_expectation.json
@@ -0,0 +1,53 @@
+[ {
+ "id" : "local-1430917381534",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-05-06T13:03:00.893GMT",
+ "endTime" : "2015-05-06T13:03:11.398GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1426533911241",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "attemptId" : "2",
+ "startTime" : "2015-03-17T23:11:50.242GMT",
+ "endTime" : "2015-03-17T23:12:25.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ }, {
+ "attemptId" : "1",
+ "startTime" : "2015-03-16T19:25:10.242GMT",
+ "endTime" : "2015-03-16T19:25:45.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1425081759269",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-28T00:02:38.277GMT",
+ "endTime" : "2015-02-28T00:02:46.912GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1422981780767",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:59.720GMT",
+ "endTime" : "2015-02-03T16:43:08.731GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1422981759269",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:38.277GMT",
+ "endTime" : "2015-02-03T16:42:46.912GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/executor_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/executor_list_json_expectation.json
new file mode 100644
index 0000000000000..cb622e147249e
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/executor_list_json_expectation.json
@@ -0,0 +1,17 @@
+[ {
+ "id" : "",
+ "hostPort" : "localhost:57971",
+ "rddBlocks" : 8,
+ "memoryUsed" : 28000128,
+ "diskUsed" : 0,
+ "activeTasks" : 0,
+ "failedTasks" : 1,
+ "completedTasks" : 31,
+ "totalTasks" : 32,
+ "totalDuration" : 8820,
+ "totalInputBytes" : 28000288,
+ "totalShuffleRead" : 0,
+ "totalShuffleWrite" : 13180,
+ "maxMemory" : 278302556,
+ "executorLogs" : { }
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/failed_stage_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/failed_stage_list_json_expectation.json
new file mode 100644
index 0000000000000..bff6a4f69d077
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/failed_stage_list_json_expectation.json
@@ -0,0 +1,23 @@
+[ {
+ "status" : "FAILED",
+ "stageId" : 2,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 7,
+ "numFailedTasks" : 1,
+ "executorRunTime" : 278,
+ "inputBytes" : 0,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 0,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "count at :20",
+ "details" : "org.apache.spark.rdd.RDD.count(RDD.scala:910)\n$line11.$read$$iwC$$iwC$$iwC$$iwC.(:20)\n$line11.$read$$iwC$$iwC$$iwC.(:25)\n$line11.$read$$iwC$$iwC.(:27)\n$line11.$read$$iwC.(:29)\n$line11.$read.(:31)\n$line11.$read$.(:35)\n$line11.$read$.()\n$line11.$eval$.(:7)\n$line11.$eval$.()\n$line11.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/job_list_from_multi_attempt_app_json_1__expectation.json b/core/src/test/resources/HistoryServerExpectations/job_list_from_multi_attempt_app_json_1__expectation.json
new file mode 100644
index 0000000000000..2e92e1fa0ec23
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/job_list_from_multi_attempt_app_json_1__expectation.json
@@ -0,0 +1,15 @@
+[ {
+ "jobId" : 0,
+ "name" : "foreach at :15",
+ "stageIds" : [ 0 ],
+ "status" : "SUCCEEDED",
+ "numTasks" : 8,
+ "numActiveTasks" : 0,
+ "numCompletedTasks" : 8,
+ "numSkippedTasks" : 8,
+ "numFailedTasks" : 0,
+ "numActiveStages" : 0,
+ "numCompletedStages" : 1,
+ "numSkippedStages" : 0,
+ "numFailedStages" : 0
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/job_list_from_multi_attempt_app_json_2__expectation.json b/core/src/test/resources/HistoryServerExpectations/job_list_from_multi_attempt_app_json_2__expectation.json
new file mode 100644
index 0000000000000..2e92e1fa0ec23
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/job_list_from_multi_attempt_app_json_2__expectation.json
@@ -0,0 +1,15 @@
+[ {
+ "jobId" : 0,
+ "name" : "foreach at :15",
+ "stageIds" : [ 0 ],
+ "status" : "SUCCEEDED",
+ "numTasks" : 8,
+ "numActiveTasks" : 0,
+ "numCompletedTasks" : 8,
+ "numSkippedTasks" : 8,
+ "numFailedTasks" : 0,
+ "numActiveStages" : 0,
+ "numCompletedStages" : 1,
+ "numSkippedStages" : 0,
+ "numFailedStages" : 0
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/job_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/job_list_json_expectation.json
new file mode 100644
index 0000000000000..cab4750270dfa
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/job_list_json_expectation.json
@@ -0,0 +1,43 @@
+[ {
+ "jobId" : 2,
+ "name" : "count at :17",
+ "stageIds" : [ 3 ],
+ "status" : "SUCCEEDED",
+ "numTasks" : 8,
+ "numActiveTasks" : 0,
+ "numCompletedTasks" : 8,
+ "numSkippedTasks" : 8,
+ "numFailedTasks" : 0,
+ "numActiveStages" : 0,
+ "numCompletedStages" : 1,
+ "numSkippedStages" : 0,
+ "numFailedStages" : 0
+}, {
+ "jobId" : 1,
+ "name" : "count at :20",
+ "stageIds" : [ 1, 2 ],
+ "status" : "FAILED",
+ "numTasks" : 16,
+ "numActiveTasks" : 0,
+ "numCompletedTasks" : 15,
+ "numSkippedTasks" : 15,
+ "numFailedTasks" : 1,
+ "numActiveStages" : 0,
+ "numCompletedStages" : 1,
+ "numSkippedStages" : 0,
+ "numFailedStages" : 1
+}, {
+ "jobId" : 0,
+ "name" : "count at :15",
+ "stageIds" : [ 0 ],
+ "status" : "SUCCEEDED",
+ "numTasks" : 8,
+ "numActiveTasks" : 0,
+ "numCompletedTasks" : 8,
+ "numSkippedTasks" : 8,
+ "numFailedTasks" : 0,
+ "numActiveStages" : 0,
+ "numCompletedStages" : 1,
+ "numSkippedStages" : 0,
+ "numFailedStages" : 0
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/maxDate2_app_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/maxDate2_app_list_json_expectation.json
new file mode 100644
index 0000000000000..483632a3956ed
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/maxDate2_app_list_json_expectation.json
@@ -0,0 +1,10 @@
+[ {
+ "id" : "local-1422981759269",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:38.277GMT",
+ "endTime" : "2015-02-03T16:42:46.912GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/maxDate_app_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/maxDate_app_list_json_expectation.json
new file mode 100644
index 0000000000000..4b85690fd9199
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/maxDate_app_list_json_expectation.json
@@ -0,0 +1,19 @@
+[ {
+ "id" : "local-1422981780767",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:59.720GMT",
+ "endTime" : "2015-02-03T16:43:08.731GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1422981759269",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:38.277GMT",
+ "endTime" : "2015-02-03T16:42:46.912GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/minDate_app_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/minDate_app_list_json_expectation.json
new file mode 100644
index 0000000000000..dca86fe5f7e6a
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/minDate_app_list_json_expectation.json
@@ -0,0 +1,35 @@
+[ {
+ "id" : "local-1430917381534",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-05-06T13:03:00.893GMT",
+ "endTime" : "2015-05-06T13:03:11.398GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1426533911241",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "attemptId" : "2",
+ "startTime" : "2015-03-17T23:11:50.242GMT",
+ "endTime" : "2015-03-17T23:12:25.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ }, {
+ "attemptId" : "1",
+ "startTime" : "2015-03-16T19:25:10.242GMT",
+ "endTime" : "2015-03-16T19:25:45.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}, {
+ "id" : "local-1425081759269",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-28T00:02:38.277GMT",
+ "endTime" : "2015-02-28T00:02:46.912GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/one_app_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/one_app_json_expectation.json
new file mode 100644
index 0000000000000..07489ad96414a
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/one_app_json_expectation.json
@@ -0,0 +1,10 @@
+{
+ "id" : "local-1422981780767",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "startTime" : "2015-02-03T16:42:59.720GMT",
+ "endTime" : "2015-02-03T16:43:08.731GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/one_app_multi_attempt_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/one_app_multi_attempt_json_expectation.json
new file mode 100644
index 0000000000000..8f3d7160c723f
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/one_app_multi_attempt_json_expectation.json
@@ -0,0 +1,17 @@
+{
+ "id" : "local-1426533911241",
+ "name" : "Spark shell",
+ "attempts" : [ {
+ "attemptId" : "2",
+ "startTime" : "2015-03-17T23:11:50.242GMT",
+ "endTime" : "2015-03-17T23:12:25.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ }, {
+ "attemptId" : "1",
+ "startTime" : "2015-03-16T19:25:10.242GMT",
+ "endTime" : "2015-03-16T19:25:45.177GMT",
+ "sparkUser" : "irashid",
+ "completed" : true
+ } ]
+}
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/one_job_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/one_job_json_expectation.json
new file mode 100644
index 0000000000000..4a29072bdb6e4
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/one_job_json_expectation.json
@@ -0,0 +1,15 @@
+{
+ "jobId" : 0,
+ "name" : "count at :15",
+ "stageIds" : [ 0 ],
+ "status" : "SUCCEEDED",
+ "numTasks" : 8,
+ "numActiveTasks" : 0,
+ "numCompletedTasks" : 8,
+ "numSkippedTasks" : 8,
+ "numFailedTasks" : 0,
+ "numActiveStages" : 0,
+ "numCompletedStages" : 1,
+ "numSkippedStages" : 0,
+ "numFailedStages" : 0
+}
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/one_rdd_storage_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/one_rdd_storage_json_expectation.json
new file mode 100644
index 0000000000000..38b5328ffbb03
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/one_rdd_storage_json_expectation.json
@@ -0,0 +1,64 @@
+{
+ "id" : 0,
+ "name" : "0",
+ "numPartitions" : 8,
+ "numCachedPartitions" : 8,
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 28000128,
+ "diskUsed" : 0,
+ "dataDistribution" : [ {
+ "address" : "localhost:57971",
+ "memoryUsed" : 28000128,
+ "memoryRemaining" : 250302428,
+ "diskUsed" : 0
+ } ],
+ "partitions" : [ {
+ "blockName" : "rdd_0_0",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ }, {
+ "blockName" : "rdd_0_1",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ }, {
+ "blockName" : "rdd_0_2",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ }, {
+ "blockName" : "rdd_0_3",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ }, {
+ "blockName" : "rdd_0_4",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ }, {
+ "blockName" : "rdd_0_5",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ }, {
+ "blockName" : "rdd_0_6",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ }, {
+ "blockName" : "rdd_0_7",
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 3500016,
+ "diskUsed" : 0,
+ "executors" : [ "localhost:57971" ]
+ } ]
+}
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/one_stage_attempt_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/one_stage_attempt_json_expectation.json
new file mode 100644
index 0000000000000..111cb8163eb3d
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/one_stage_attempt_json_expectation.json
@@ -0,0 +1,270 @@
+{
+ "status" : "COMPLETE",
+ "stageId" : 1,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 3476,
+ "inputBytes" : 28000128,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 13180,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "map at :14",
+ "details" : "org.apache.spark.rdd.RDD.map(RDD.scala:271)\n$line10.$read$$iwC$$iwC$$iwC$$iwC.(:14)\n$line10.$read$$iwC$$iwC$$iwC.(:19)\n$line10.$read$$iwC$$iwC.(:21)\n$line10.$read$$iwC.(:23)\n$line10.$read.(:25)\n$line10.$read$.(:29)\n$line10.$read$.()\n$line10.$eval$.(:7)\n$line10.$eval$.()\n$line10.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ],
+ "tasks" : {
+ "8" : {
+ "taskId" : 8,
+ "index" : 0,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.829GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 1,
+ "executorRunTime" : 435,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 2,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 94000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "11" : {
+ "taskId" : 11,
+ "index" : 3,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.830GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1647,
+ "writeTime" : 83000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "14" : {
+ "taskId" : 14,
+ "index" : 6,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.832GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 88000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "13" : {
+ "taskId" : 13,
+ "index" : 5,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.831GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 2,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 73000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "10" : {
+ "taskId" : 10,
+ "index" : 2,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.830GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 76000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "9" : {
+ "taskId" : 9,
+ "index" : 1,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.830GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 1,
+ "executorRunTime" : 436,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 98000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "12" : {
+ "taskId" : 12,
+ "index" : 4,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.831GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1645,
+ "writeTime" : 101000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "15" : {
+ "taskId" : 15,
+ "index" : 7,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.833GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 1,
+ "executorRunTime" : 435,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 79000,
+ "recordsWritten" : 0
+ }
+ }
+ }
+ },
+ "executorSummary" : {
+ "" : {
+ "taskTime" : 3624,
+ "failedTasks" : 0,
+ "succeededTasks" : 8,
+ "inputBytes" : 28000128,
+ "outputBytes" : 0,
+ "shuffleRead" : 0,
+ "shuffleWrite" : 13180,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0
+ }
+ }
+}
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/one_stage_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/one_stage_json_expectation.json
new file mode 100644
index 0000000000000..ef339f89afa45
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/one_stage_json_expectation.json
@@ -0,0 +1,270 @@
+[ {
+ "status" : "COMPLETE",
+ "stageId" : 1,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 3476,
+ "inputBytes" : 28000128,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 13180,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "map at :14",
+ "details" : "org.apache.spark.rdd.RDD.map(RDD.scala:271)\n$line10.$read$$iwC$$iwC$$iwC$$iwC.(:14)\n$line10.$read$$iwC$$iwC$$iwC.(:19)\n$line10.$read$$iwC$$iwC.(:21)\n$line10.$read$$iwC.(:23)\n$line10.$read.(:25)\n$line10.$read$.(:29)\n$line10.$read$.()\n$line10.$eval$.(:7)\n$line10.$eval$.()\n$line10.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ],
+ "tasks" : {
+ "8" : {
+ "taskId" : 8,
+ "index" : 0,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.829GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 1,
+ "executorRunTime" : 435,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 2,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 94000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "11" : {
+ "taskId" : 11,
+ "index" : 3,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.830GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1647,
+ "writeTime" : 83000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "14" : {
+ "taskId" : 14,
+ "index" : 6,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.832GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 88000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "13" : {
+ "taskId" : 13,
+ "index" : 5,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.831GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 2,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 73000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "10" : {
+ "taskId" : 10,
+ "index" : 2,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.830GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 76000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "9" : {
+ "taskId" : 9,
+ "index" : 1,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.830GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 1,
+ "executorRunTime" : 436,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 98000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "12" : {
+ "taskId" : 12,
+ "index" : 4,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.831GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 2,
+ "executorRunTime" : 434,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1645,
+ "writeTime" : 101000,
+ "recordsWritten" : 0
+ }
+ }
+ },
+ "15" : {
+ "taskId" : 15,
+ "index" : 7,
+ "attempt" : 0,
+ "launchTime" : "2015-02-03T16:43:05.833GMT",
+ "executorId" : "",
+ "host" : "localhost",
+ "taskLocality" : "PROCESS_LOCAL",
+ "speculative" : false,
+ "accumulatorUpdates" : [ ],
+ "taskMetrics" : {
+ "executorDeserializeTime" : 1,
+ "executorRunTime" : 435,
+ "resultSize" : 1902,
+ "jvmGcTime" : 19,
+ "resultSerializationTime" : 1,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "inputMetrics" : {
+ "bytesRead" : 3500016,
+ "recordsRead" : 0
+ },
+ "shuffleWriteMetrics" : {
+ "bytesWritten" : 1648,
+ "writeTime" : 79000,
+ "recordsWritten" : 0
+ }
+ }
+ }
+ },
+ "executorSummary" : {
+ "" : {
+ "taskTime" : 3624,
+ "failedTasks" : 0,
+ "succeededTasks" : 8,
+ "inputBytes" : 28000128,
+ "outputBytes" : 0,
+ "shuffleRead" : 0,
+ "shuffleWrite" : 13180,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0
+ }
+ }
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/rdd_list_storage_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/rdd_list_storage_json_expectation.json
new file mode 100644
index 0000000000000..f79a31022d214
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/rdd_list_storage_json_expectation.json
@@ -0,0 +1,9 @@
+[ {
+ "id" : 0,
+ "name" : "0",
+ "numPartitions" : 8,
+ "numCachedPartitions" : 8,
+ "storageLevel" : "Memory Deserialized 1x Replicated",
+ "memoryUsed" : 28000128,
+ "diskUsed" : 0
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/running_app_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/running_app_list_json_expectation.json
new file mode 100644
index 0000000000000..8878e547a7984
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/running_app_list_json_expectation.json
@@ -0,0 +1 @@
+[ ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/stage_list_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_list_json_expectation.json
new file mode 100644
index 0000000000000..056fac7088594
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/stage_list_json_expectation.json
@@ -0,0 +1,89 @@
+[ {
+ "status" : "COMPLETE",
+ "stageId" : 3,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 162,
+ "inputBytes" : 160,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 0,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "count at :17",
+ "details" : "org.apache.spark.rdd.RDD.count(RDD.scala:910)\n$line19.$read$$iwC$$iwC$$iwC$$iwC.(:17)\n$line19.$read$$iwC$$iwC$$iwC.(:22)\n$line19.$read$$iwC$$iwC.(:24)\n$line19.$read$$iwC.(:26)\n$line19.$read.(:28)\n$line19.$read$.(:32)\n$line19.$read$.()\n$line19.$eval$.(:7)\n$line19.$eval$.()\n$line19.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+}, {
+ "status" : "COMPLETE",
+ "stageId" : 1,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 3476,
+ "inputBytes" : 28000128,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 13180,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "map at :14",
+ "details" : "org.apache.spark.rdd.RDD.map(RDD.scala:271)\n$line10.$read$$iwC$$iwC$$iwC$$iwC.(:14)\n$line10.$read$$iwC$$iwC$$iwC.(:19)\n$line10.$read$$iwC$$iwC.(:21)\n$line10.$read$$iwC.(:23)\n$line10.$read.(:25)\n$line10.$read$.(:29)\n$line10.$read$.()\n$line10.$eval$.(:7)\n$line10.$eval$.()\n$line10.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+}, {
+ "status" : "COMPLETE",
+ "stageId" : 0,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 4338,
+ "inputBytes" : 0,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 0,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "count at :15",
+ "details" : "org.apache.spark.rdd.RDD.count(RDD.scala:910)\n$line9.$read$$iwC$$iwC$$iwC$$iwC.(:15)\n$line9.$read$$iwC$$iwC$$iwC.(:20)\n$line9.$read$$iwC$$iwC.(:22)\n$line9.$read$$iwC.(:24)\n$line9.$read.(:26)\n$line9.$read$.(:30)\n$line9.$read$.()\n$line9.$eval$.(:7)\n$line9.$eval$.()\n$line9.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+}, {
+ "status" : "FAILED",
+ "stageId" : 2,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 7,
+ "numFailedTasks" : 1,
+ "executorRunTime" : 278,
+ "inputBytes" : 0,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 0,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "count at :20",
+ "details" : "org.apache.spark.rdd.RDD.count(RDD.scala:910)\n$line11.$read$$iwC$$iwC$$iwC$$iwC.(:20)\n$line11.$read$$iwC$$iwC$$iwC.(:25)\n$line11.$read$$iwC$$iwC.(:27)\n$line11.$read$$iwC.(:29)\n$line11.$read.(:31)\n$line11.$read$.(:35)\n$line11.$read$.()\n$line11.$eval$.(:7)\n$line11.$eval$.()\n$line11.$eval.$print()\nsun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)\nsun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:57)\nsun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)\njava.lang.reflect.Method.invoke(Method.java:606)\norg.apache.spark.repl.SparkIMain$ReadEvalPrint.call(SparkIMain.scala:852)\norg.apache.spark.repl.SparkIMain$Request.loadAndRun(SparkIMain.scala:1125)\norg.apache.spark.repl.SparkIMain.loadAndRunReq$1(SparkIMain.scala:674)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:705)\norg.apache.spark.repl.SparkIMain.interpret(SparkIMain.scala:669)",
+ "schedulingPool" : "default",
+ "accumulatorUpdates" : [ ]
+} ]
\ No newline at end of file
diff --git a/core/src/test/resources/HistoryServerExpectations/stage_list_with_accumulable_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_list_with_accumulable_json_expectation.json
new file mode 100644
index 0000000000000..79ccacd309693
--- /dev/null
+++ b/core/src/test/resources/HistoryServerExpectations/stage_list_with_accumulable_json_expectation.json
@@ -0,0 +1,27 @@
+[ {
+ "status" : "COMPLETE",
+ "stageId" : 0,
+ "attemptId" : 0,
+ "numActiveTasks" : 0,
+ "numCompleteTasks" : 8,
+ "numFailedTasks" : 0,
+ "executorRunTime" : 120,
+ "inputBytes" : 0,
+ "inputRecords" : 0,
+ "outputBytes" : 0,
+ "outputRecords" : 0,
+ "shuffleReadBytes" : 0,
+ "shuffleReadRecords" : 0,
+ "shuffleWriteBytes" : 0,
+ "shuffleWriteRecords" : 0,
+ "memoryBytesSpilled" : 0,
+ "diskBytesSpilled" : 0,
+ "name" : "foreach at :15",
+ "details" : "org.apache.spark.rdd.RDD.foreach(RDD.scala:765)\n$line9.$read$$iwC$$iwC$$iwC$$iwC.(:15)\n$line9.$read$$iwC$$iwC$$iwC.(:20)\n$line9.$read$$iwC$$iwC.(:22)\n$line9.$read$$iwC.(:24)\n$line9.$read.(:26)\n$line9.$read$.(:30)\n$line9.$read$.()\n$line9.$eval$.