Skip to content
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,8 @@ package org.apache.spark.sql.execution.streaming.state

import java.io._

import scala.util.control.NonFatal

import org.apache.hadoop.conf.Configuration

import org.apache.spark.{SparkConf, SparkEnv}
Expand Down Expand Up @@ -233,7 +235,15 @@ private[sql] class RocksDBStateStoreProvider
}

override def doMaintenance(): Unit = {
rocksDB.doMaintenance()
try {
rocksDB.doMaintenance()
} catch {
// SPARK-46547 - Swallow non-fatal exception in maintenance task to avoid deadlock between
// maintenance thread and streaming aggregation operator
case NonFatal(ex) =>
logWarning(s"Ignoring error while performing maintenance operations with exception=",
ex)
}
}

override def close(): Unit = {
Expand Down