@@ -26,6 +26,8 @@ import scala.ref.WeakReference
2626import scala .util .Try
2727
2828import org .apache .hadoop .conf .Configuration
29+ import org .json4s .NoTypeHints
30+ import org .json4s .jackson .Serialization
2931import org .rocksdb .{RocksDB => NativeRocksDB , _ }
3032
3133import org .apache .spark .TaskContext
@@ -72,6 +74,7 @@ class RocksDB(
7274 dbOptions.setTableFormatConfig(tableFormatConfig)
7375 private val dbLogger = createLogger() // for forwarding RocksDB native logs to log4j
7476 dbOptions.setStatistics(new Statistics ())
77+ private val nativeStats = dbOptions.statistics()
7578
7679 private val workingDir = createTempDir(" workingDir" )
7780 private val fileManager = new RocksDBFileManager (
@@ -84,6 +87,8 @@ class RocksDB(
8487 @ volatile private var loadedVersion = - 1L // -1 = nothing valid is loaded
8588 @ volatile private var numKeysOnLoadedVersion = 0L
8689 @ volatile private var numKeysOnWritingVersion = 0L
90+ @ volatile private var fileManagerMetrics = RocksDBFileManagerMetrics .EMPTY_METRICS
91+ @ volatile private var acquiredThreadInfo : AcquiredThreadInfo = _
8792
8893 @ GuardedBy (" acquireLock" )
8994 @ volatile private var acquiredThreadInfo : AcquiredThreadInfo = _
@@ -105,6 +110,7 @@ class RocksDB(
105110 numKeysOnWritingVersion = metadata.numKeys
106111 numKeysOnLoadedVersion = metadata.numKeys
107112 loadedVersion = version
113+ fileManagerMetrics = fileManager.latestloadCheckpointMetrics
108114 }
109115 writeBatch.clear()
110116 logInfo(s " Loaded $version" )
@@ -223,6 +229,7 @@ class RocksDB(
223229 }
224230 numKeysOnLoadedVersion = numKeysOnWritingVersion
225231 loadedVersion = newVersion
232+ fileManagerMetrics = fileManager.latestSaveCheckpointMetrics
226233 commitLatencyMs ++= Map (
227234 " writeBatch" -> writeTimeMs,
228235 " flush" -> flushTimeMs,
@@ -231,6 +238,7 @@ class RocksDB(
231238 " checkpoint" -> checkpointTimeMs,
232239 " fileSync" -> fileSyncTimeMs
233240 )
241+ logInfo(s " Committed $newVersion, stats = ${metrics.json}" )
234242 loadedVersion
235243 } catch {
236244 case t : Throwable =>
@@ -283,6 +291,30 @@ class RocksDB(
283291 /** Get the latest version available in the DFS */
284292 def getLatestVersion (): Long = fileManager.getLatestVersion()
285293
294+ /** Get current instantaneous statistics */
295+ def metrics : RocksDBMetrics = {
296+ import HistogramType ._
297+ val totalSSTFilesBytes = getDBProperty(" rocksdb.total-sst-files-size" )
298+ val readerMemUsage = getDBProperty(" rocksdb.estimate-table-readers-mem" )
299+ val memTableMemUsage = getDBProperty(" rocksdb.size-all-mem-tables" )
300+ val nativeOps = Seq (" get" -> DB_GET , " put" -> DB_WRITE ).toMap
301+ val nativeOpsLatencyMicros = nativeOps.mapValues { typ =>
302+ RocksDBNativeHistogram (nativeStats.getHistogramData(typ))
303+ }
304+
305+ RocksDBMetrics (
306+ numCommittedKeys,
307+ numUncommittedKeys,
308+ readerMemUsage + memTableMemUsage,
309+ totalSSTFilesBytes,
310+ nativeOpsLatencyMicros,
311+ commitLatencyMs,
312+ bytesCopied = fileManagerMetrics.bytesCopied,
313+ filesCopied = fileManagerMetrics.filesCopied,
314+ filesReused = fileManagerMetrics.filesReused,
315+ zipFileBytesUncompressed = fileManagerMetrics.zipFileBytesUncompressed)
316+ }
317+
286318 private def acquire (): Unit = acquireLock.synchronized {
287319 val newAcquiredThreadInfo = AcquiredThreadInfo ()
288320 val waitStartTime = System .currentTimeMillis
@@ -388,7 +420,6 @@ class ByteArrayPair(var key: Array[Byte] = null, var value: Array[Byte] = null)
388420
389421/**
390422 * Configurations for optimizing RocksDB
391- *
392423 * @param compactOnCommit Whether to compact RocksDB data before commit / checkpointing
393424 */
394425case class RocksDBConf (
@@ -442,6 +473,42 @@ object RocksDBConf {
442473 def apply (): RocksDBConf = apply(new StateStoreConf ())
443474}
444475
476+ /** Class to represent stats from each commit. */
477+ case class RocksDBMetrics (
478+ numCommittedKeys : Long ,
479+ numUncommittedKeys : Long ,
480+ memUsageBytes : Long ,
481+ totalSSTFilesBytes : Long ,
482+ nativeOpsLatencyMicros : Map [String , RocksDBNativeHistogram ],
483+ lastCommitLatencyMs : Map [String , Long ],
484+ filesCopied : Long ,
485+ bytesCopied : Long ,
486+ filesReused : Long ,
487+ zipFileBytesUncompressed : Option [Long ]) {
488+ def json : String = Serialization .write(this )(RocksDBMetrics .format)
489+ }
490+
491+ object RocksDBMetrics {
492+ val format = Serialization .formats(NoTypeHints )
493+ }
494+
495+ /** Class to wrap RocksDB's native histogram */
496+ case class RocksDBNativeHistogram (
497+ avg : Double , stddev : Double , median : Double , p95 : Double , p99 : Double ) {
498+ def json : String = Serialization .write(this )(RocksDBMetrics .format)
499+ }
500+
501+ object RocksDBNativeHistogram {
502+ def apply (nativeHist : HistogramData ): RocksDBNativeHistogram = {
503+ RocksDBNativeHistogram (
504+ nativeHist.getAverage,
505+ nativeHist.getStandardDeviation,
506+ nativeHist.getMedian,
507+ nativeHist.getPercentile95,
508+ nativeHist.getPercentile99)
509+ }
510+ }
511+
445512case class AcquiredThreadInfo () {
446513 val threadRef : WeakReference [Thread ] = new WeakReference [Thread ](Thread .currentThread())
447514 val tc : TaskContext = TaskContext .get()
0 commit comments