This is an automated email from the ASF dual-hosted git repository.

kabhwan pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/spark.git


The following commit(s) were added to refs/heads/master by this push:
     new b42ac52e787 [SPARK-45582][SS] Ensure that store instance is not used 
after calling commit within output mode streaming aggregation
b42ac52e787 is described below

commit b42ac52e787c896f21452023da0bd6685a1b47fc
Author: Anish Shrigondekar <anish.shrigonde...@databricks.com>
AuthorDate: Thu Oct 19 06:51:45 2023 +0900

    [SPARK-45582][SS] Ensure that store instance is not used after calling 
commit within output mode streaming aggregation
    
    ### What changes were proposed in this pull request?
    Ensure that store instance is not used after calling commit within output 
mode streaming aggregation
    
    ### Why are the changes needed?
    Without these changes, we were accessing the store instance to retrieve the 
iterator even after the commit was called. When commit is called, we release 
the DB instance lock. So its possible task retries can acquire the instance 
lock and close the DB instance. So when the original thread tries to access the 
DB, it might run into a null pointer exception. This change fixes the issue
    
    ```
            org.apache.spark.sql.streaming.StreamingQueryException: Job aborted 
due to stage failure: Task 0 in stage 5.0 failed 1 times, most recent failure: 
Lost task 0.0 in stage 5.0 (TID 492) 
(ip-10-110-25-116.us-west-2.compute.internal executor driver): 
java.lang.NullPointerException
            at 
org.apache.spark.sql.execution.streaming.state.RocksDB.iterator(RocksDB.scala:337)
            at 
org.apache.spark.sql.execution.streaming.state.RocksDBStateStoreProvider$RocksDBStateStore.iterator(RocksDBStateStoreProvider.scala:79)
            at 
org.apache.spark.sql.execution.streaming.state.StreamingAggregationStateManagerImplV1.values(StreamingAggregationStateManager.scala:130)
            at 
org.apache.spark.sql.execution.streaming.StateStoreSaveExec.$anonfun$doExecute$5(statefulOperators.scala:543)
            at 
org.apache.spark.sql.execution.streaming.state.package$StateStoreOps.$anonfun$mapPartitionsWithStateStore$1(package.scala:63)
            at 
org.apache.spark.sql.execution.streaming.state.StateStoreRDD.compute(StateStoreRDD.scala:131)
            at 
org.apache.spark.rdd.RDD.$anonfun$computeOrReadCheckpoint$1(RDD.scala:407)
            at 
com.databricks.spark.util.ExecutorFrameProfiler$.record(ExecutorFrameProfiler.scala:110)
            at org.apache.spark.rdd.RDD.computeOrReadCheckpoint(RDD.scala:404)
            at org.apache.spark.rdd.RDD.iterator(RDD.scala:371)
    ```
    
    ### Does this PR introduce _any_ user-facing change?
    No
    
    ### How was this patch tested?
    RocksDBStreamingAggregationSuite
    ```
    18:12:00.242 WARN 
org.apache.spark.sql.streaming.RocksDBStateStoreStreamingAggregationSuite:
    
    ===== POSSIBLE THREAD LEAK IN SUITE 
o.a.s.sql.streaming.RocksDBStateStoreStreamingAggregationSuite, threads: 
ForkJoinPool.commonPool-worker-6 (daemon=true), 
ForkJoinPool.commonPool-worker-4 (daemon=true), 
ForkJoinPool.commonPool-worker-7 (daemon=true), 
ForkJoinPool.commonPool-worker-5 (daemon=true), 
ForkJoinPool.commonPool-worker-3 (daemon=true), state-store-maintenance-task 
(daemon=true), rpc-boss-3-1 (daemon=true), ForkJoinPool.commonPool-worker-8 
(daemon=true), shuffle-boss-6-1 (da [...]
    [info] Run completed in 5 minutes, 8 seconds.
    [info] Total number of tests run: 80
    [info] Suites: completed 1, aborted 0
    [info] Tests: succeeded 80, failed 0, canceled 0, ignored 0, pending 0
    [info] All tests passed.
    ```
    
    StreamingSessionWindowSuite
    ```
    ===== POSSIBLE THREAD LEAK IN SUITE 
o.a.s.sql.streaming.StreamingSessionWindowSuite, threads: 
ForkJoinPool.commonPool-worker-6 (daemon=true), 
state-store-maintenance-thread-0 (daemon=true), 
ForkJoinPool.commonPool-worker-4 (daemon=true), 
state-store-maintenance-thread-1 (daemon=true), 
ForkJoinPool.commonPool-worker-7 (daemon=true), 
ForkJoinPool.commonPool-worker-5 (daemon=true), 
ForkJoinPool.commonPool-worker-3 (daemon=true), state-store-maintenance-task 
(daemon=true), rpc-boss-3-1 (d [...]
    [info] Run completed in 3 minutes, 38 seconds.
    [info] Total number of tests run: 48
    [info] Suites: completed 1, aborted 0
    [info] Tests: succeeded 48, failed 0, canceled 0, ignored 0, pending 0
    [info] All tests passed.
    ```
    
    ### Was this patch authored or co-authored using generative AI tooling?
    No
    
    Closes #43413 from anishshri-db/task/SPARK-45582.
    
    Authored-by: Anish Shrigondekar <anish.shrigonde...@databricks.com>
    Signed-off-by: Jungtaek Lim <kabhwan.opensou...@gmail.com>
---
 .../execution/streaming/statefulOperators.scala    | 64 ++++++++++++++++------
 1 file changed, 48 insertions(+), 16 deletions(-)

diff --git 
a/sql/core/src/main/scala/org/apache/spark/sql/execution/streaming/statefulOperators.scala
 
b/sql/core/src/main/scala/org/apache/spark/sql/execution/streaming/statefulOperators.scala
index f534c5f108a..bda560b1721 100644
--- 
a/sql/core/src/main/scala/org/apache/spark/sql/execution/streaming/statefulOperators.scala
+++ 
b/sql/core/src/main/scala/org/apache/spark/sql/execution/streaming/statefulOperators.scala
@@ -515,15 +515,31 @@ case class StateStoreSaveExec(
                 numUpdatedStateRows += 1
               }
             }
-            allRemovalsTimeMs += 0
-            commitTimeMs += timeTakenMs {
-              stateManager.commit(store)
-            }
-            setStoreMetrics(store)
-            setOperatorMetrics()
-            stateManager.values(store).map { valueRow =>
-              numOutputRows += 1
-              valueRow
+
+            // SPARK-45582 - Ensure that store instance is not used after 
commit is called
+            // to invoke the iterator.
+            val rangeIter = stateManager.values(store)
+
+            new NextIterator[InternalRow] {
+              override protected def getNext(): InternalRow = {
+                if (rangeIter.hasNext) {
+                  val valueRow = rangeIter.next()
+                  numOutputRows += 1
+                  valueRow
+                } else {
+                  finished = true
+                  null
+                }
+              }
+
+              override protected def close(): Unit = {
+                allRemovalsTimeMs += 0
+                commitTimeMs += timeTakenMs {
+                  stateManager.commit(store)
+                }
+                setStoreMetrics(store)
+                setOperatorMetrics()
+              }
             }
 
           // Update and output only rows being evicted from the StateStore
@@ -782,13 +798,29 @@ case class SessionWindowStateStoreSaveExec(
           allUpdatesTimeMs += timeTakenMs {
             putToStore(iter, store)
           }
-          commitTimeMs += timeTakenMs {
-            stateManager.commit(store)
-          }
-          setStoreMetrics(store)
-          stateManager.iterator(store).map { row =>
-            numOutputRows += 1
-            row
+
+          // SPARK-45582 - Ensure that store instance is not used after commit 
is called
+          // to invoke the iterator.
+          val rangeIter = stateManager.iterator(store)
+
+          new NextIterator[InternalRow] {
+            override protected def getNext(): InternalRow = {
+              if (rangeIter.hasNext) {
+                val valueRow = rangeIter.next()
+                numOutputRows += 1
+                valueRow
+              } else {
+                finished = true
+                null
+              }
+            }
+
+            override protected def close(): Unit = {
+              commitTimeMs += timeTakenMs {
+                stateManager.commit(store)
+              }
+              setStoreMetrics(store)
+            }
           }
 
         // Update and output only rows being evicted from the StateStore


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscr...@spark.apache.org
For additional commands, e-mail: commits-h...@spark.apache.org

Reply via email to