@@ -25,7 +25,6 @@ import org.apache.spark.streaming.scheduler.StreamingListenerBatchStarted
2525import org .apache .spark .streaming .scheduler .BatchInfo
2626import org .apache .spark .streaming .scheduler .StreamingListenerBatchSubmitted
2727import org .apache .spark .util .Distribution
28- import org .apache .spark .Logging
2928
3029
3130private [streaming] class StreamingJobProgressListener (ssc : StreamingContext )
@@ -37,6 +36,7 @@ private[streaming] class StreamingJobProgressListener(ssc: StreamingContext)
3736 private val batchInfoLimit = ssc.conf.getInt(" spark.streaming.ui.retainedBatches" , 100 )
3837 private var totalCompletedBatches = 0L
3938 private val receiverInfos = new HashMap [Int , ReceiverInfo ]
39+ private var totalReceivedBatchRecords = 0L
4040
4141 val batchDuration = ssc.graph.batchDuration.milliseconds
4242
@@ -65,6 +65,10 @@ private[streaming] class StreamingJobProgressListener(ssc: StreamingContext)
6565 override def onBatchStarted (batchStarted : StreamingListenerBatchStarted ) = synchronized {
6666 runningBatchInfos(batchStarted.batchInfo.batchTime) = batchStarted.batchInfo
6767 waitingBatchInfos.remove(batchStarted.batchInfo.batchTime)
68+
69+ batchStarted.batchInfo.receivedBlockInfo.foreach { case (_, infos) =>
70+ totalReceivedBatchRecords += infos.map(_.numRecords).sum
71+ }
6872 }
6973
7074 override def onBatchCompleted (batchCompleted : StreamingListenerBatchCompleted ) = synchronized {
@@ -83,6 +87,10 @@ private[streaming] class StreamingJobProgressListener(ssc: StreamingContext)
8387 totalCompletedBatches
8488 }
8589
90+ def numTotalReceivedBatchRecords : Long = synchronized {
91+ totalReceivedBatchRecords
92+ }
93+
8694 def numUnprocessedBatches : Long = synchronized {
8795 waitingBatchInfos.size + runningBatchInfos.size
8896 }
0 commit comments