Skip to content
Snippets Groups Projects
Commit 4cff0b50 authored by Liwei Lin's avatar Liwei Lin Committed by Shixiong Zhu
Browse files

[SPARK-19168][STRUCTURED STREAMING] StateStore should be aborted upon error


## What changes were proposed in this pull request?

We should call `StateStore.abort()` when there should be any error before the store is committed.

## How was this patch tested?

Manually.

Author: Liwei Lin <lwlin7@gmail.com>

Closes #16547 from lw-lin/append-filter.

(cherry picked from commit 569e5068)
Signed-off-by: default avatarShixiong Zhu <shixiong@databricks.com>
parent 047506ba
No related branches found
No related tags found
No related merge requests found
......@@ -31,6 +31,7 @@ import org.apache.spark.sql.execution.streaming.state._
import org.apache.spark.sql.execution.SparkPlan
import org.apache.spark.sql.streaming.OutputMode
import org.apache.spark.sql.types.StructType
import org.apache.spark.TaskContext
/** Used to identify the state store for a given operator. */
......@@ -150,6 +151,13 @@ case class StateStoreSaveExec(
val numTotalStateRows = longMetric("numTotalStateRows")
val numUpdatedStateRows = longMetric("numUpdatedStateRows")
// Abort the state store in case of error
TaskContext.get().addTaskCompletionListener(_ => {
if (!store.hasCommitted) {
store.abort()
}
})
outputMode match {
// Update and output all rows in the StateStore.
case Some(Complete) =>
......
......@@ -203,7 +203,7 @@ private[state] class HDFSBackedStateStoreProvider(
/**
* Whether all updates have been committed
*/
override private[state] def hasCommitted: Boolean = {
override private[streaming] def hasCommitted: Boolean = {
state == COMMITTED
}
......
......@@ -83,7 +83,7 @@ trait StateStore {
/**
* Whether all updates have been committed
*/
private[state] def hasCommitted: Boolean
private[streaming] def hasCommitted: Boolean
}
......
0% Loading or .
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment