From ae859fb9b944d679de3367b14d9f70838b55914b Mon Sep 17 00:00:00 2001 From: jinxing Date: Fri, 9 Jun 2017 08:59:29 +0800 Subject: [PATCH] [SPARK-19937] Collect metrics for remote bytes read to disk during shuffle. --- .../apache/spark/InternalAccumulator.scala | 1 + .../spark/executor/ShuffleReadMetrics.scala | 13 +++++ .../apache/spark/executor/TaskMetrics.scala | 1 + .../status/api/v1/AllStagesResource.scala | 2 + .../org/apache/spark/status/api/v1/api.scala | 2 + .../storage/ShuffleBlockFetcherIterator.scala | 6 +++ .../org/apache/spark/ui/jobs/UIData.scala | 4 +- .../org/apache/spark/util/JsonProtocol.scala | 3 ++ .../one_stage_attempt_json_expectation.json | 8 +++ .../one_stage_json_expectation.json | 8 +++ .../stage_task_list_expectation.json | 20 ++++++++ ...multi_attempt_app_json_1__expectation.json | 8 +++ ...multi_attempt_app_json_2__expectation.json | 8 +++ ...k_list_w__offset___length_expectation.json | 50 +++++++++++++++++++ ...stage_task_list_w__sortBy_expectation.json | 20 ++++++++ ...tBy_short_names___runtime_expectation.json | 20 ++++++++ ...rtBy_short_names__runtime_expectation.json | 20 ++++++++ ...mmary_w__custom_quantiles_expectation.json | 1 + ...sk_summary_w_shuffle_read_expectation.json | 1 + ...k_summary_w_shuffle_write_expectation.json | 1 + ...age_with_accumulable_json_expectation.json | 8 +++ .../spark/executor/TaskMetricsSuite.scala | 3 ++ .../apache/spark/util/JsonProtocolSuite.scala | 33 ++++++++---- project/MimaExcludes.scala | 4 ++ 24 files changed, 233 insertions(+), 12 deletions(-) diff --git a/core/src/main/scala/org/apache/spark/InternalAccumulator.scala b/core/src/main/scala/org/apache/spark/InternalAccumulator.scala index 82d3098e2e055..18b10d23da94c 100644 --- a/core/src/main/scala/org/apache/spark/InternalAccumulator.scala +++ b/core/src/main/scala/org/apache/spark/InternalAccumulator.scala @@ -50,6 +50,7 @@ private[spark] object InternalAccumulator { val REMOTE_BLOCKS_FETCHED = SHUFFLE_READ_METRICS_PREFIX + "remoteBlocksFetched" val LOCAL_BLOCKS_FETCHED = SHUFFLE_READ_METRICS_PREFIX + "localBlocksFetched" val REMOTE_BYTES_READ = SHUFFLE_READ_METRICS_PREFIX + "remoteBytesRead" + val REMOTE_BYTES_READ_TO_DISK = SHUFFLE_READ_METRICS_PREFIX + "remoteBytesReadToDisk" val LOCAL_BYTES_READ = SHUFFLE_READ_METRICS_PREFIX + "localBytesRead" val FETCH_WAIT_TIME = SHUFFLE_READ_METRICS_PREFIX + "fetchWaitTime" val RECORDS_READ = SHUFFLE_READ_METRICS_PREFIX + "recordsRead" diff --git a/core/src/main/scala/org/apache/spark/executor/ShuffleReadMetrics.scala b/core/src/main/scala/org/apache/spark/executor/ShuffleReadMetrics.scala index 8dd1a1ea059be..4be395c8358b2 100644 --- a/core/src/main/scala/org/apache/spark/executor/ShuffleReadMetrics.scala +++ b/core/src/main/scala/org/apache/spark/executor/ShuffleReadMetrics.scala @@ -31,6 +31,7 @@ class ShuffleReadMetrics private[spark] () extends Serializable { private[executor] val _remoteBlocksFetched = new LongAccumulator private[executor] val _localBlocksFetched = new LongAccumulator private[executor] val _remoteBytesRead = new LongAccumulator + private[executor] val _remoteBytesReadToDisk = new LongAccumulator private[executor] val _localBytesRead = new LongAccumulator private[executor] val _fetchWaitTime = new LongAccumulator private[executor] val _recordsRead = new LongAccumulator @@ -50,6 +51,11 @@ class ShuffleReadMetrics private[spark] () extends Serializable { */ def remoteBytesRead: Long = _remoteBytesRead.sum + /** + * Total number of remotes bytes read to disk from the shuffle by this task. + */ + def remoteBytesReadToDisk: Long = _remoteBytesReadToDisk.sum + /** * Shuffle data that was read from the local disk (as opposed to from a remote executor). */ @@ -80,6 +86,7 @@ class ShuffleReadMetrics private[spark] () extends Serializable { private[spark] def incRemoteBlocksFetched(v: Long): Unit = _remoteBlocksFetched.add(v) private[spark] def incLocalBlocksFetched(v: Long): Unit = _localBlocksFetched.add(v) private[spark] def incRemoteBytesRead(v: Long): Unit = _remoteBytesRead.add(v) + private[spark] def incRemoteBytesReadToDisk(v: Long): Unit = _remoteBytesReadToDisk.add(v) private[spark] def incLocalBytesRead(v: Long): Unit = _localBytesRead.add(v) private[spark] def incFetchWaitTime(v: Long): Unit = _fetchWaitTime.add(v) private[spark] def incRecordsRead(v: Long): Unit = _recordsRead.add(v) @@ -87,6 +94,7 @@ class ShuffleReadMetrics private[spark] () extends Serializable { private[spark] def setRemoteBlocksFetched(v: Int): Unit = _remoteBlocksFetched.setValue(v) private[spark] def setLocalBlocksFetched(v: Int): Unit = _localBlocksFetched.setValue(v) private[spark] def setRemoteBytesRead(v: Long): Unit = _remoteBytesRead.setValue(v) + private[spark] def setRemoteBytesReadToDisk(v: Long): Unit = _remoteBytesReadToDisk.setValue(v) private[spark] def setLocalBytesRead(v: Long): Unit = _localBytesRead.setValue(v) private[spark] def setFetchWaitTime(v: Long): Unit = _fetchWaitTime.setValue(v) private[spark] def setRecordsRead(v: Long): Unit = _recordsRead.setValue(v) @@ -99,6 +107,7 @@ class ShuffleReadMetrics private[spark] () extends Serializable { _remoteBlocksFetched.setValue(0) _localBlocksFetched.setValue(0) _remoteBytesRead.setValue(0) + _remoteBytesReadToDisk.setValue(0) _localBytesRead.setValue(0) _fetchWaitTime.setValue(0) _recordsRead.setValue(0) @@ -106,6 +115,7 @@ class ShuffleReadMetrics private[spark] () extends Serializable { _remoteBlocksFetched.add(metric.remoteBlocksFetched) _localBlocksFetched.add(metric.localBlocksFetched) _remoteBytesRead.add(metric.remoteBytesRead) + _remoteBytesReadToDisk.add(metric.remoteBytesReadToDisk) _localBytesRead.add(metric.localBytesRead) _fetchWaitTime.add(metric.fetchWaitTime) _recordsRead.add(metric.recordsRead) @@ -122,6 +132,7 @@ private[spark] class TempShuffleReadMetrics { private[this] var _remoteBlocksFetched = 0L private[this] var _localBlocksFetched = 0L private[this] var _remoteBytesRead = 0L + private[this] var _remoteBytesReadToDisk = 0L private[this] var _localBytesRead = 0L private[this] var _fetchWaitTime = 0L private[this] var _recordsRead = 0L @@ -129,6 +140,7 @@ private[spark] class TempShuffleReadMetrics { def incRemoteBlocksFetched(v: Long): Unit = _remoteBlocksFetched += v def incLocalBlocksFetched(v: Long): Unit = _localBlocksFetched += v def incRemoteBytesRead(v: Long): Unit = _remoteBytesRead += v + def incRemoteBytesReadToDisk(v: Long): Unit = _remoteBytesReadToDisk += v def incLocalBytesRead(v: Long): Unit = _localBytesRead += v def incFetchWaitTime(v: Long): Unit = _fetchWaitTime += v def incRecordsRead(v: Long): Unit = _recordsRead += v @@ -136,6 +148,7 @@ private[spark] class TempShuffleReadMetrics { def remoteBlocksFetched: Long = _remoteBlocksFetched def localBlocksFetched: Long = _localBlocksFetched def remoteBytesRead: Long = _remoteBytesRead + def remoteBytesReadToDisk: Long = _remoteBytesReadToDisk def localBytesRead: Long = _localBytesRead def fetchWaitTime: Long = _fetchWaitTime def recordsRead: Long = _recordsRead diff --git a/core/src/main/scala/org/apache/spark/executor/TaskMetrics.scala b/core/src/main/scala/org/apache/spark/executor/TaskMetrics.scala index a3ce3d1ccc5e3..341a6da8107ef 100644 --- a/core/src/main/scala/org/apache/spark/executor/TaskMetrics.scala +++ b/core/src/main/scala/org/apache/spark/executor/TaskMetrics.scala @@ -215,6 +215,7 @@ class TaskMetrics private[spark] () extends Serializable { shuffleRead.REMOTE_BLOCKS_FETCHED -> shuffleReadMetrics._remoteBlocksFetched, shuffleRead.LOCAL_BLOCKS_FETCHED -> shuffleReadMetrics._localBlocksFetched, shuffleRead.REMOTE_BYTES_READ -> shuffleReadMetrics._remoteBytesRead, + shuffleRead.REMOTE_BYTES_READ_TO_DISK -> shuffleReadMetrics._remoteBytesReadToDisk, shuffleRead.LOCAL_BYTES_READ -> shuffleReadMetrics._localBytesRead, shuffleRead.FETCH_WAIT_TIME -> shuffleReadMetrics._fetchWaitTime, shuffleRead.RECORDS_READ -> shuffleReadMetrics._recordsRead, diff --git a/core/src/main/scala/org/apache/spark/status/api/v1/AllStagesResource.scala b/core/src/main/scala/org/apache/spark/status/api/v1/AllStagesResource.scala index 1818935392eb3..56028710ecc66 100644 --- a/core/src/main/scala/org/apache/spark/status/api/v1/AllStagesResource.scala +++ b/core/src/main/scala/org/apache/spark/status/api/v1/AllStagesResource.scala @@ -200,6 +200,7 @@ private[v1] object AllStagesResource { readBytes = submetricQuantiles(_.totalBytesRead), readRecords = submetricQuantiles(_.recordsRead), remoteBytesRead = submetricQuantiles(_.remoteBytesRead), + remoteBytesReadToDisk = submetricQuantiles(_.remoteBytesReadToDisk), remoteBlocksFetched = submetricQuantiles(_.remoteBlocksFetched), localBlocksFetched = submetricQuantiles(_.localBlocksFetched), totalBlocksFetched = submetricQuantiles(_.totalBlocksFetched), @@ -281,6 +282,7 @@ private[v1] object AllStagesResource { localBlocksFetched = internal.localBlocksFetched, fetchWaitTime = internal.fetchWaitTime, remoteBytesRead = internal.remoteBytesRead, + remoteBytesReadToDisk = internal.remoteBytesReadToDisk, localBytesRead = internal.localBytesRead, recordsRead = internal.recordsRead ) diff --git a/core/src/main/scala/org/apache/spark/status/api/v1/api.scala b/core/src/main/scala/org/apache/spark/status/api/v1/api.scala index f6203271f3cd2..05948f2661056 100644 --- a/core/src/main/scala/org/apache/spark/status/api/v1/api.scala +++ b/core/src/main/scala/org/apache/spark/status/api/v1/api.scala @@ -208,6 +208,7 @@ class ShuffleReadMetrics private[spark]( val localBlocksFetched: Long, val fetchWaitTime: Long, val remoteBytesRead: Long, + val remoteBytesReadToDisk: Long, val localBytesRead: Long, val recordsRead: Long) @@ -249,6 +250,7 @@ class ShuffleReadMetricDistributions private[spark]( val localBlocksFetched: IndexedSeq[Double], val fetchWaitTime: IndexedSeq[Double], val remoteBytesRead: IndexedSeq[Double], + val remoteBytesReadToDisk: IndexedSeq[Double], val totalBlocksFetched: IndexedSeq[Double]) class ShuffleWriteMetricDistributions private[spark]( diff --git a/core/src/main/scala/org/apache/spark/storage/ShuffleBlockFetcherIterator.scala b/core/src/main/scala/org/apache/spark/storage/ShuffleBlockFetcherIterator.scala index bded3a1e4eb54..a10f1feadd0af 100644 --- a/core/src/main/scala/org/apache/spark/storage/ShuffleBlockFetcherIterator.scala +++ b/core/src/main/scala/org/apache/spark/storage/ShuffleBlockFetcherIterator.scala @@ -165,6 +165,9 @@ final class ShuffleBlockFetcherIterator( case SuccessFetchResult(_, address, _, buf, _) => if (address != blockManager.blockManagerId) { shuffleMetrics.incRemoteBytesRead(buf.size) + if (buf.isInstanceOf[FileSegmentManagedBuffer]) { + shuffleMetrics.incRemoteBytesReadToDisk(buf.size) + } shuffleMetrics.incRemoteBlocksFetched(1) } buf.release() @@ -363,6 +366,9 @@ final class ShuffleBlockFetcherIterator( case r @ SuccessFetchResult(blockId, address, size, buf, isNetworkReqDone) => if (address != blockManager.blockManagerId) { shuffleMetrics.incRemoteBytesRead(buf.size) + if (buf.isInstanceOf[FileSegmentManagedBuffer]) { + shuffleMetrics.incRemoteBytesReadToDisk(buf.size) + } shuffleMetrics.incRemoteBlocksFetched(1) } bytesInFlight -= size diff --git a/core/src/main/scala/org/apache/spark/ui/jobs/UIData.scala b/core/src/main/scala/org/apache/spark/ui/jobs/UIData.scala index 6764daa0df529..9448baac096dc 100644 --- a/core/src/main/scala/org/apache/spark/ui/jobs/UIData.scala +++ b/core/src/main/scala/org/apache/spark/ui/jobs/UIData.scala @@ -251,6 +251,7 @@ private[spark] object UIData { remoteBlocksFetched: Long, localBlocksFetched: Long, remoteBytesRead: Long, + remoteBytesReadToDisk: Long, localBytesRead: Long, fetchWaitTime: Long, recordsRead: Long, @@ -274,6 +275,7 @@ private[spark] object UIData { remoteBlocksFetched = metrics.remoteBlocksFetched, localBlocksFetched = metrics.localBlocksFetched, remoteBytesRead = metrics.remoteBytesRead, + remoteBytesReadToDisk = metrics.remoteBytesReadToDisk, localBytesRead = metrics.localBytesRead, fetchWaitTime = metrics.fetchWaitTime, recordsRead = metrics.recordsRead, @@ -282,7 +284,7 @@ private[spark] object UIData { ) } } - private val EMPTY = ShuffleReadMetricsUIData(0, 0, 0, 0, 0, 0, 0, 0) + private val EMPTY = ShuffleReadMetricsUIData(0, 0, 0, 0, 0, 0, 0, 0, 0) } case class ShuffleWriteMetricsUIData( diff --git a/core/src/main/scala/org/apache/spark/util/JsonProtocol.scala b/core/src/main/scala/org/apache/spark/util/JsonProtocol.scala index 8296c4294242c..806d14e7cc119 100644 --- a/core/src/main/scala/org/apache/spark/util/JsonProtocol.scala +++ b/core/src/main/scala/org/apache/spark/util/JsonProtocol.scala @@ -339,6 +339,7 @@ private[spark] object JsonProtocol { ("Local Blocks Fetched" -> taskMetrics.shuffleReadMetrics.localBlocksFetched) ~ ("Fetch Wait Time" -> taskMetrics.shuffleReadMetrics.fetchWaitTime) ~ ("Remote Bytes Read" -> taskMetrics.shuffleReadMetrics.remoteBytesRead) ~ + ("Remote Bytes Read To Disk" -> taskMetrics.shuffleReadMetrics.remoteBytesReadToDisk) ~ ("Local Bytes Read" -> taskMetrics.shuffleReadMetrics.localBytesRead) ~ ("Total Records Read" -> taskMetrics.shuffleReadMetrics.recordsRead) val shuffleWriteMetrics: JValue = @@ -804,6 +805,8 @@ private[spark] object JsonProtocol { readMetrics.incRemoteBlocksFetched((readJson \ "Remote Blocks Fetched").extract[Int]) readMetrics.incLocalBlocksFetched((readJson \ "Local Blocks Fetched").extract[Int]) readMetrics.incRemoteBytesRead((readJson \ "Remote Bytes Read").extract[Long]) + Utils.jsonOption(readJson \ "Remote Bytes Read To Disk") + .foreach { v => readMetrics.incRemoteBytesReadToDisk(v.extract[Long])} readMetrics.incLocalBytesRead( Utils.jsonOption(readJson \ "Local Bytes Read").map(_.extract[Long]).getOrElse(0L)) readMetrics.incFetchWaitTime((readJson \ "Fetch Wait Time").extract[Long]) 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 index c2f450ba87c6d..6fb40f6f1713b 100644 --- a/core/src/test/resources/HistoryServerExpectations/one_stage_attempt_json_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/one_stage_attempt_json_expectation.json @@ -60,6 +60,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -105,6 +106,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -150,6 +152,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -195,6 +198,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -240,6 +244,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -285,6 +290,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -330,6 +336,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -375,6 +382,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/one_stage_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/one_stage_json_expectation.json index 506859ae545b1..f5a89a2107646 100644 --- a/core/src/test/resources/HistoryServerExpectations/one_stage_json_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/one_stage_json_expectation.json @@ -60,6 +60,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -105,6 +106,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -150,6 +152,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -195,6 +198,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -240,6 +244,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -285,6 +290,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -330,6 +336,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -375,6 +382,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_list_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_list_expectation.json index f4cec68fbfdf2..9b401b414f8d4 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_list_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_list_expectation.json @@ -33,6 +33,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -77,6 +78,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -121,6 +123,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -165,6 +168,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -209,6 +213,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -253,6 +258,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -297,6 +303,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -341,6 +348,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -385,6 +393,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -429,6 +438,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -473,6 +483,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -517,6 +528,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -561,6 +573,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -605,6 +618,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -649,6 +663,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -693,6 +708,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -737,6 +753,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -781,6 +798,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -825,6 +843,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -869,6 +888,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_1__expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_1__expectation.json index 496a21c328da9..2ebee66a6d7c2 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_1__expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_1__expectation.json @@ -38,6 +38,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -87,6 +88,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -136,6 +138,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -185,6 +188,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -234,6 +238,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -283,6 +288,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -332,6 +338,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -381,6 +388,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_2__expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_2__expectation.json index 4328dc753c5d4..965a31a4104c3 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_2__expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_list_from_multi_attempt_app_json_2__expectation.json @@ -38,6 +38,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -87,6 +88,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -136,6 +138,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -185,6 +188,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -234,6 +238,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -283,6 +288,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -332,6 +338,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -381,6 +388,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__offset___length_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__offset___length_expectation.json index 8c571430f3a1f..31132e156937c 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__offset___length_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__offset___length_expectation.json @@ -33,6 +33,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -77,6 +78,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -121,6 +123,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -165,6 +168,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -209,6 +213,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -253,6 +258,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -297,6 +303,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -341,6 +348,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -385,6 +393,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -429,6 +438,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -473,6 +483,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -517,6 +528,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -561,6 +573,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -605,6 +618,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -649,6 +663,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -693,6 +708,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -737,6 +753,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -781,6 +798,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -825,6 +843,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -869,6 +888,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -913,6 +933,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -957,6 +978,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1001,6 +1023,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1045,6 +1068,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1089,6 +1113,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1133,6 +1158,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1177,6 +1203,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1221,6 +1248,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1265,6 +1293,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1309,6 +1338,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1353,6 +1383,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1397,6 +1428,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1441,6 +1473,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1485,6 +1518,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1529,6 +1563,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1573,6 +1608,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1617,6 +1653,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1661,6 +1698,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1705,6 +1743,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1749,6 +1788,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1793,6 +1833,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1837,6 +1878,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1881,6 +1923,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1925,6 +1968,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -1969,6 +2013,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -2013,6 +2058,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -2057,6 +2103,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -2101,6 +2148,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -2145,6 +2193,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -2189,6 +2238,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_expectation.json index 0bd614bdc756e..6af1cfbeb8f7e 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_expectation.json @@ -33,6 +33,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -77,6 +78,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -121,6 +123,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -165,6 +168,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -209,6 +213,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -253,6 +258,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -297,6 +303,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -341,6 +348,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -385,6 +393,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -429,6 +438,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -473,6 +483,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -517,6 +528,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -561,6 +573,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -605,6 +618,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -649,6 +663,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -693,6 +708,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -737,6 +753,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -781,6 +798,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -825,6 +843,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -869,6 +888,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names___runtime_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names___runtime_expectation.json index 0bd614bdc756e..6af1cfbeb8f7e 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names___runtime_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names___runtime_expectation.json @@ -33,6 +33,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -77,6 +78,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -121,6 +123,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -165,6 +168,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -209,6 +213,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -253,6 +258,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -297,6 +303,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -341,6 +348,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -385,6 +393,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -429,6 +438,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -473,6 +483,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -517,6 +528,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -561,6 +573,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -605,6 +618,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -649,6 +663,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -693,6 +708,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -737,6 +753,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -781,6 +798,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -825,6 +843,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -869,6 +888,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names__runtime_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names__runtime_expectation.json index b58f1a51ba481..c26daf4b8d7bd 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names__runtime_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_list_w__sortBy_short_names__runtime_expectation.json @@ -33,6 +33,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -77,6 +78,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -121,6 +123,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -165,6 +168,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -209,6 +213,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -253,6 +258,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -297,6 +303,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -341,6 +348,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -385,6 +393,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -429,6 +438,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -473,6 +483,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -517,6 +528,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -561,6 +573,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -605,6 +618,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -649,6 +663,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -693,6 +708,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -737,6 +753,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -781,6 +798,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -825,6 +843,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -869,6 +888,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w__custom_quantiles_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w__custom_quantiles_expectation.json index 0ed609d5b7f92..f8e27703c0def 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w__custom_quantiles_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w__custom_quantiles_expectation.json @@ -24,6 +24,7 @@ "localBlocksFetched" : [ 0.0, 0.0, 0.0 ], "fetchWaitTime" : [ 0.0, 0.0, 0.0 ], "remoteBytesRead" : [ 0.0, 0.0, 0.0 ], + "remoteBytesReadToDisk" : [ 0.0, 0.0, 0.0 ], "totalBlocksFetched" : [ 0.0, 0.0, 0.0 ] }, "shuffleWriteMetrics" : { diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_read_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_read_expectation.json index 6d230ac653776..a28bda16a956e 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_read_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_read_expectation.json @@ -24,6 +24,7 @@ "localBlocksFetched" : [ 100.0, 100.0, 100.0, 100.0, 100.0 ], "fetchWaitTime" : [ 0.0, 0.0, 0.0, 1.0, 1.0 ], "remoteBytesRead" : [ 0.0, 0.0, 0.0, 0.0, 0.0 ], + "remoteBytesReadToDisk" : [ 0.0, 0.0, 0.0, 0.0, 0.0 ], "totalBlocksFetched" : [ 100.0, 100.0, 100.0, 100.0, 100.0 ] }, "shuffleWriteMetrics" : { diff --git a/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_write_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_write_expectation.json index aea0f5413d8b9..ede3eaed1d1d2 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_write_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_task_summary_w_shuffle_write_expectation.json @@ -24,6 +24,7 @@ "localBlocksFetched" : [ 0.0, 0.0, 0.0, 0.0, 0.0 ], "fetchWaitTime" : [ 0.0, 0.0, 0.0, 0.0, 0.0 ], "remoteBytesRead" : [ 0.0, 0.0, 0.0, 0.0, 0.0 ], + "remoteBytesReadToDisk" : [ 0.0, 0.0, 0.0, 0.0, 0.0 ], "totalBlocksFetched" : [ 0.0, 0.0, 0.0, 0.0, 0.0 ] }, "shuffleWriteMetrics" : { diff --git a/core/src/test/resources/HistoryServerExpectations/stage_with_accumulable_json_expectation.json b/core/src/test/resources/HistoryServerExpectations/stage_with_accumulable_json_expectation.json index a449926ee7dc6..44b5f66efe339 100644 --- a/core/src/test/resources/HistoryServerExpectations/stage_with_accumulable_json_expectation.json +++ b/core/src/test/resources/HistoryServerExpectations/stage_with_accumulable_json_expectation.json @@ -69,6 +69,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -119,6 +120,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -169,6 +171,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -219,6 +222,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -269,6 +273,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -319,6 +324,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -369,6 +375,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, @@ -419,6 +426,7 @@ "localBlocksFetched" : 0, "fetchWaitTime" : 0, "remoteBytesRead" : 0, + "remoteBytesReadToDisk" : 0, "localBytesRead" : 0, "recordsRead" : 0 }, diff --git a/core/src/test/scala/org/apache/spark/executor/TaskMetricsSuite.scala b/core/src/test/scala/org/apache/spark/executor/TaskMetricsSuite.scala index eae26fa742a23..7bcc2fb5231db 100644 --- a/core/src/test/scala/org/apache/spark/executor/TaskMetricsSuite.scala +++ b/core/src/test/scala/org/apache/spark/executor/TaskMetricsSuite.scala @@ -94,6 +94,8 @@ class TaskMetricsSuite extends SparkFunSuite { sr.setRemoteBytesRead(30L) sr.incRemoteBytesRead(3L) sr.incRemoteBytesRead(3L) + sr.setRemoteBytesReadToDisk(10L) + sr.incRemoteBytesReadToDisk(8L) sr.setLocalBytesRead(400L) sr.setLocalBytesRead(40L) sr.incLocalBytesRead(4L) @@ -110,6 +112,7 @@ class TaskMetricsSuite extends SparkFunSuite { assert(sr.remoteBlocksFetched == 12) assert(sr.localBlocksFetched == 24) assert(sr.remoteBytesRead == 36L) + assert(sr.remoteBytesReadToDisk == 18L) assert(sr.localBytesRead == 48L) assert(sr.fetchWaitTime == 60L) assert(sr.recordsRead == 72L) diff --git a/core/src/test/scala/org/apache/spark/util/JsonProtocolSuite.scala b/core/src/test/scala/org/apache/spark/util/JsonProtocolSuite.scala index a77c8e3cab4e8..57452d4912abe 100644 --- a/core/src/test/scala/org/apache/spark/util/JsonProtocolSuite.scala +++ b/core/src/test/scala/org/apache/spark/util/JsonProtocolSuite.scala @@ -848,6 +848,7 @@ private[spark] object JsonProtocolSuite extends Assertions { } else { val sr = t.createTempShuffleReadMetrics() sr.incRemoteBytesRead(b + d) + sr.incRemoteBytesReadToDisk(b) sr.incLocalBlocksFetched(e) sr.incFetchWaitTime(a + d) sr.incRemoteBlocksFetched(f) @@ -1128,6 +1129,7 @@ private[spark] object JsonProtocolSuite extends Assertions { | "Local Blocks Fetched": 700, | "Fetch Wait Time": 900, | "Remote Bytes Read": 1000, + | "Remote Bytes Read To Disk": 400, | "Local Bytes Read": 1100, | "Total Records Read": 10 | }, @@ -1228,6 +1230,7 @@ private[spark] object JsonProtocolSuite extends Assertions { | "Local Blocks Fetched" : 0, | "Fetch Wait Time" : 0, | "Remote Bytes Read" : 0, + | "Remote Bytes Read To Disk" : 0, | "Local Bytes Read" : 0, | "Total Records Read" : 0 | }, @@ -1328,10 +1331,11 @@ private[spark] object JsonProtocolSuite extends Assertions { | "Local Blocks Fetched" : 0, | "Fetch Wait Time" : 0, | "Remote Bytes Read" : 0, + | "Remote Bytes Read To Disk" : 0, | "Local Bytes Read" : 0, | "Total Records Read" : 0 | }, - | "Shuffle Write Metrics" : { + | "Shuffle Write Metrics": { | "Shuffle Bytes Written" : 0, | "Shuffle Write Time" : 0, | "Shuffle Records Written" : 0 @@ -1915,76 +1919,83 @@ private[spark] object JsonProtocolSuite extends Assertions { | }, | { | "ID": 14, - | "Name": "${shuffleRead.LOCAL_BYTES_READ}", + | "Name": "${shuffleRead.REMOTE_BYTES_READ_TO_DISK}", | "Update": 0, | "Internal": true, | "Count Failed Values": true | }, | { | "ID": 15, - | "Name": "${shuffleRead.FETCH_WAIT_TIME}", + | "Name": "${shuffleRead.LOCAL_BYTES_READ}", | "Update": 0, | "Internal": true, | "Count Failed Values": true | }, | { | "ID": 16, - | "Name": "${shuffleRead.RECORDS_READ}", + | "Name": "${shuffleRead.FETCH_WAIT_TIME}", | "Update": 0, | "Internal": true, | "Count Failed Values": true | }, | { | "ID": 17, - | "Name": "${shuffleWrite.BYTES_WRITTEN}", + | "Name": "${shuffleRead.RECORDS_READ}", | "Update": 0, | "Internal": true, | "Count Failed Values": true | }, | { | "ID": 18, - | "Name": "${shuffleWrite.RECORDS_WRITTEN}", + | "Name": "${shuffleWrite.BYTES_WRITTEN}", | "Update": 0, | "Internal": true, | "Count Failed Values": true | }, | { | "ID": 19, - | "Name": "${shuffleWrite.WRITE_TIME}", + | "Name": "${shuffleWrite.RECORDS_WRITTEN}", | "Update": 0, | "Internal": true, | "Count Failed Values": true | }, | { | "ID": 20, + | "Name": "${shuffleWrite.WRITE_TIME}", + | "Update": 0, + | "Internal": true, + | "Count Failed Values": true + | }, + | { + | "ID": 21, | "Name": "${input.BYTES_READ}", | "Update": 2100, | "Internal": true, | "Count Failed Values": true | }, | { - | "ID": 21, + | "ID": 22, | "Name": "${input.RECORDS_READ}", | "Update": 21, | "Internal": true, | "Count Failed Values": true | }, | { - | "ID": 22, + | "ID": 23, | "Name": "${output.BYTES_WRITTEN}", | "Update": 1200, | "Internal": true, | "Count Failed Values": true | }, | { - | "ID": 23, + | "ID": 24, | "Name": "${output.RECORDS_WRITTEN}", | "Update": 12, | "Internal": true, | "Count Failed Values": true | }, | { - | "ID": 24, + | "ID": 25, | "Name": "$TEST_ACCUM", | "Update": 0, | "Internal": true, diff --git a/project/MimaExcludes.scala b/project/MimaExcludes.scala index 3cc089dcede38..2670038d424d0 100644 --- a/project/MimaExcludes.scala +++ b/project/MimaExcludes.scala @@ -38,6 +38,10 @@ object MimaExcludes { lazy val v23excludes = v22excludes ++ Seq( // [SPARK-20495][SQL] Add StorageLevel to cacheTable API ProblemFilters.exclude[ReversedMissingMethodProblem]("org.apache.spark.sql.catalog.Catalog.cacheTable") + ) ++ Seq( + // [SPARK-19937] Add remote bytes read to disk. + ProblemFilters.exclude[DirectMissingMethodProblem]("org.apache.spark.status.api.v1.ShuffleReadMetrics.this"), + ProblemFilters.exclude[DirectMissingMethodProblem]("org.apache.spark.status.api.v1.ShuffleReadMetricDistributions.this") ) // Exclude rules for 2.2.x