Package org.apache.spark.sql.streaming
Interface StatefulProcessorHandle
- All Superinterfaces:
Serializable
Represents the operation handle provided to the stateful processor used in the arbitrary state
API v2.
-
Method Summary
Modifier and TypeMethodDescriptionvoid
deleteIfExists
(String stateName) Function to delete and purge state variable if defined previouslyvoid
deleteTimer
(long expiryTimestampMs) Function to delete a processing/event time based timer for given implicit grouping key and provided timestamp<T> ListState<T>
getListState
(String stateName, Encoder<T> valEncoder) Creates new or returns existing list state associated with stateName.<T> ListState<T>
getListState
(String stateName, Encoder<T> valEncoder, TTLConfig ttlConfig) Function to create new or return existing list state variable of given type with ttl.<K,
V> MapState<K, V> getMapState
(String stateName, Encoder<K> userKeyEnc, Encoder<V> valEncoder) Creates new or returns existing map state associated with stateName.<K,
V> MapState<K, V> getMapState
(String stateName, Encoder<K> userKeyEnc, Encoder<V> valEncoder, TTLConfig ttlConfig) Function to create new or return existing map state variable of given type with ttl.Function to return queryInfo for currently running task<T> ValueState<T>
getValueState
(String stateName, Encoder<T> valEncoder) Function to create new or return existing single value state variable of given type.<T> ValueState<T>
getValueState
(String stateName, Encoder<T> valEncoder, TTLConfig ttlConfig) Function to create new or return existing single value state variable of given type with ttl.scala.collection.Iterator<Object>
Function to list all the timers registered for given implicit grouping key Note: calling listTimers() within thehandleInputRows
method of the StatefulProcessor will return all the unprocessed registered timers, including the one being fired within the invocation ofhandleInputRows
.void
registerTimer
(long expiryTimestampMs) Function to register a processing/event time based timer for given implicit grouping key and provided timestamp
-
Method Details
-
deleteIfExists
Function to delete and purge state variable if defined previously- Parameters:
stateName
- \- name of the state variable
-
deleteTimer
void deleteTimer(long expiryTimestampMs) Function to delete a processing/event time based timer for given implicit grouping key and provided timestamp- Parameters:
expiryTimestampMs
- \- timer expiry timestamp in milliseconds
-
getListState
Creates new or returns existing list state associated with stateName. The ListState persists values of type T.- Parameters:
stateName
- \- name of the state variablevalEncoder
- \- SQL encoder for state variable- Returns:
- \- instance of ListState of type T that can be used to store state persistently
-
getListState
Function to create new or return existing list state variable of given type with ttl. State values will not be returned past ttlDuration, and will be eventually removed from the state store. Any values in listState which have expired after ttlDuration will not be returned on get() and will be eventually removed from the state.The user must ensure to call this function only within the
init()
method of the StatefulProcessor.- Parameters:
stateName
- \- name of the state variablevalEncoder
- \- SQL encoder for state variablettlConfig
- \- the ttl configuration (time to live duration etc.)- Returns:
- \- instance of ListState of type T that can be used to store state persistently
-
getMapState
Creates new or returns existing map state associated with stateName. The MapState persists Key-Value pairs of type [K, V].- Parameters:
stateName
- \- name of the state variableuserKeyEnc
- \- spark sql encoder for the map keyvalEncoder
- \- spark sql encoder for the map value- Returns:
- \- instance of MapState of type [K,V] that can be used to store state persistently
-
getMapState
<K,V> MapState<K,V> getMapState(String stateName, Encoder<K> userKeyEnc, Encoder<V> valEncoder, TTLConfig ttlConfig) Function to create new or return existing map state variable of given type with ttl. State values will not be returned past ttlDuration, and will be eventually removed from the state store. Any values in mapState which have expired after ttlDuration will not returned on get() and will be eventually removed from the state.The user must ensure to call this function only within the
init()
method of the StatefulProcessor.- Parameters:
stateName
- \- name of the state variableuserKeyEnc
- \- spark sql encoder for the map keyvalEncoder
- \- SQL encoder for state variablettlConfig
- \- the ttl configuration (time to live duration etc.)- Returns:
- \- instance of MapState of type [K,V] that can be used to store state persistently
-
getQueryInfo
QueryInfo getQueryInfo()Function to return queryInfo for currently running task -
getValueState
Function to create new or return existing single value state variable of given type. The user must ensure to call this function only within theinit()
method of the StatefulProcessor.- Parameters:
stateName
- \- name of the state variablevalEncoder
- \- SQL encoder for state variable- Returns:
- \- instance of ValueState of type T that can be used to store state persistently
-
getValueState
Function to create new or return existing single value state variable of given type with ttl. State values will not be returned past ttlDuration, and will be eventually removed from the state store. Any state update resets the ttl to current processing time plus ttlDuration.The user must ensure to call this function only within the
init()
method of the StatefulProcessor.- Parameters:
stateName
- \- name of the state variablevalEncoder
- \- SQL encoder for state variablettlConfig
- \- the ttl configuration (time to live duration etc.)- Returns:
- \- instance of ValueState of type T that can be used to store state persistently
-
listTimers
scala.collection.Iterator<Object> listTimers()Function to list all the timers registered for given implicit grouping key Note: calling listTimers() within thehandleInputRows
method of the StatefulProcessor will return all the unprocessed registered timers, including the one being fired within the invocation ofhandleInputRows
.- Returns:
- \- list of all the registered timers for given implicit grouping key
-
registerTimer
void registerTimer(long expiryTimestampMs) Function to register a processing/event time based timer for given implicit grouping key and provided timestamp- Parameters:
expiryTimestampMs
- \- timer expiry timestamp in milliseconds
-