Package org.apache.flink.state.api
Class SavepointReader
- java.lang.Object
-
- org.apache.flink.state.api.SavepointReader
-
@PublicEvolving public class SavepointReader extends Object
The entry point for reading state from a Flink savepoint.
-
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description static SavepointReader
read(StreamExecutionEnvironment env, String path)
Loads an existing savepoint.static SavepointReader
read(StreamExecutionEnvironment env, String path, StateBackend stateBackend)
Loads an existing savepoint.<K,V>
DataStream<Tuple2<K,V>>readBroadcastState(OperatorIdentifier identifier, String name, TypeInformation<K> keyTypeInfo, TypeInformation<V> valueTypeInfo)
Read operatorBroadcastState
from aSavepoint
.<K,V>
DataStream<Tuple2<K,V>>readBroadcastState(OperatorIdentifier identifier, String name, TypeInformation<K> keyTypeInfo, TypeInformation<V> valueTypeInfo, TypeSerializer<K> keySerializer, TypeSerializer<V> valueSerializer)
Read operatorBroadcastState
from aSavepoint
when a custom serializer was used; e.g., a different serializer than the one returned byTypeInformation#createSerializer
.<K,OUT>
DataStream<OUT>readKeyedState(OperatorIdentifier identifier, KeyedStateReaderFunction<K,OUT> function)
Read keyed state from an operator in aSavepoint
.<K,OUT>
DataStream<OUT>readKeyedState(OperatorIdentifier identifier, KeyedStateReaderFunction<K,OUT> function, TypeInformation<K> keyTypeInfo, TypeInformation<OUT> outTypeInfo)
Read keyed state from an operator in aSavepoint
.<T> DataStream<T>
readListState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo)
Read operatorListState
from aSavepoint
.<T> DataStream<T>
readListState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo, TypeSerializer<T> serializer)
Read operatorListState
from aSavepoint
when a custom serializer was used; e.g., a different serializer than the one returned byTypeInformation#createSerializer
.<T> DataStream<T>
readUnionState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo)
Read operatorUnionState
from aSavepoint
.<T> DataStream<T>
readUnionState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo, TypeSerializer<T> serializer)
Read operatorUnionState
from aSavepoint
when a custom serializer was used; e.g., a different serializer than the one returned byTypeInformation#createSerializer
.<W extends Window>
WindowSavepointReader<W>window(TypeSerializer<W> windowSerializer)
Read window state from an operator in aSavepoint
.<W extends Window>
WindowSavepointReader<W>window(WindowAssigner<?,W> assigner)
Read window state from an operator in aSavepoint
.
-
-
-
Method Detail
-
read
public static SavepointReader read(StreamExecutionEnvironment env, String path) throws IOException
Loads an existing savepoint. Useful if you want to query the state of an existing application. The savepoint will be read using the state backend defined via the clusters configuration.- Parameters:
env
- The execution environment used to transform the savepoint.path
- The path to an existing savepoint on disk.- Returns:
- A
SavepointReader
. - Throws:
IOException
-
read
public static SavepointReader read(StreamExecutionEnvironment env, String path, StateBackend stateBackend) throws IOException
Loads an existing savepoint. Useful if you want to query the state of an existing application.- Parameters:
env
- The execution environment used to transform the savepoint.path
- The path to an existing savepoint on disk.stateBackend
- The state backend of the savepoint.- Returns:
- A
SavepointReader
. - Throws:
IOException
-
readListState
public <T> DataStream<T> readListState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo) throws IOException
Read operatorListState
from aSavepoint
.- Type Parameters:
T
- The type of the values that are in the list state.- Parameters:
identifier
- The identifier of the operator.name
- The (unique) name for the state.typeInfo
- The type of the elements in the state.- Returns:
- A
DataStream
representing the elements in state. - Throws:
IOException
- If the savepoint path is invalid or the uid does not exist.
-
readListState
public <T> DataStream<T> readListState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo, TypeSerializer<T> serializer) throws IOException
Read operatorListState
from aSavepoint
when a custom serializer was used; e.g., a different serializer than the one returned byTypeInformation#createSerializer
.- Type Parameters:
T
- The type of the values that are in the list state.- Parameters:
identifier
- The identifier of the operator.name
- The (unique) name for the state.typeInfo
- The type of the elements in the state.serializer
- The serializer used to write the elements into state.- Returns:
- A
DataStream
representing the elements in state. - Throws:
IOException
- If the savepoint path is invalid or the uid does not exist.
-
readUnionState
public <T> DataStream<T> readUnionState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo) throws IOException
Read operatorUnionState
from aSavepoint
.- Type Parameters:
T
- The type of the values that are in the union state.- Parameters:
identifier
- The identifier of the operator.name
- The (unique) name for the state.typeInfo
- The type of the elements in the state.- Returns:
- A
DataStream
representing the elements in state. - Throws:
IOException
- If the savepoint path is invalid or the uid does not exist.
-
readUnionState
public <T> DataStream<T> readUnionState(OperatorIdentifier identifier, String name, TypeInformation<T> typeInfo, TypeSerializer<T> serializer) throws IOException
Read operatorUnionState
from aSavepoint
when a custom serializer was used; e.g., a different serializer than the one returned byTypeInformation#createSerializer
.- Type Parameters:
T
- The type of the values that are in the union state.- Parameters:
identifier
- The identifier of the operator.name
- The (unique) name for the state.typeInfo
- The type of the elements in the state.serializer
- The serializer used to write the elements into state.- Returns:
- A
DataStream
representing the elements in state. - Throws:
IOException
- If the savepoint path is invalid or the uid does not exist.
-
readBroadcastState
public <K,V> DataStream<Tuple2<K,V>> readBroadcastState(OperatorIdentifier identifier, String name, TypeInformation<K> keyTypeInfo, TypeInformation<V> valueTypeInfo) throws IOException
Read operatorBroadcastState
from aSavepoint
.- Type Parameters:
K
- The type of keys in state.V
- The type of values in state.- Parameters:
identifier
- The identifier of the operator.name
- The (unique) name for the state.keyTypeInfo
- The type information for the keys in the state.valueTypeInfo
- The type information for the values in the state.- Returns:
- A
DataStream
of key-value pairs from state. - Throws:
IOException
- If the savepoint does not contain the specified uid.
-
readBroadcastState
public <K,V> DataStream<Tuple2<K,V>> readBroadcastState(OperatorIdentifier identifier, String name, TypeInformation<K> keyTypeInfo, TypeInformation<V> valueTypeInfo, TypeSerializer<K> keySerializer, TypeSerializer<V> valueSerializer) throws IOException
Read operatorBroadcastState
from aSavepoint
when a custom serializer was used; e.g., a different serializer than the one returned byTypeInformation#createSerializer
.- Type Parameters:
K
- The type of keys in state.V
- The type of values in state.- Parameters:
identifier
- The identifier of the operator.name
- The (unique) name for the state.keyTypeInfo
- The type information for the keys in the state.valueTypeInfo
- The type information for the values in the state.keySerializer
- The type serializer used to write keys into the state.valueSerializer
- The type serializer used to write values into the state.- Returns:
- A
DataStream
of key-value pairs from state. - Throws:
IOException
- If the savepoint path is invalid or the uid does not exist.
-
readKeyedState
public <K,OUT> DataStream<OUT> readKeyedState(OperatorIdentifier identifier, KeyedStateReaderFunction<K,OUT> function) throws IOException
Read keyed state from an operator in aSavepoint
.- Type Parameters:
K
- The type of the key in state.OUT
- The output type of the transform function.- Parameters:
identifier
- The identifier of the operator.function
- TheKeyedStateReaderFunction
that is called for each key in state.- Returns:
- A
DataStream
of objects read from keyed state. - Throws:
IOException
- If the savepoint does not contain operator state with the given uid.
-
readKeyedState
public <K,OUT> DataStream<OUT> readKeyedState(OperatorIdentifier identifier, KeyedStateReaderFunction<K,OUT> function, TypeInformation<K> keyTypeInfo, TypeInformation<OUT> outTypeInfo) throws IOException
Read keyed state from an operator in aSavepoint
.- Type Parameters:
K
- The type of the key in state.OUT
- The output type of the transform function.- Parameters:
identifier
- The identifier of the operator.function
- TheKeyedStateReaderFunction
that is called for each key in state.keyTypeInfo
- The type information of the key in state.outTypeInfo
- The type information of the output of the transform reader function.- Returns:
- A
DataStream
of objects read from keyed state. - Throws:
IOException
- If the savepoint does not contain operator state with the given uid.
-
window
public <W extends Window> WindowSavepointReader<W> window(WindowAssigner<?,W> assigner)
Read window state from an operator in aSavepoint
. This method supports reading from any type of window.- Parameters:
assigner
- TheWindowAssigner
used to write out the operator.- Returns:
- A
WindowSavepointReader
.
-
window
public <W extends Window> WindowSavepointReader<W> window(TypeSerializer<W> windowSerializer)
Read window state from an operator in aSavepoint
. This method supports reading from any type of window.- Parameters:
windowSerializer
- The serializer used for the window type.- Returns:
- A
WindowSavepointReader
.
-
-