Class SavepointReader


  • @PublicEvolving
    public class SavepointReader
    extends Object
    The entry point for reading state from a Flink savepoint.
    • 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
      • readListState

        public <T> DataStream<T> readListState​(OperatorIdentifier identifier,
                                               String name,
                                               TypeInformation<T> typeInfo)
                                        throws IOException
        Read operator ListState from a Savepoint.
        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 operator ListState from a Savepoint when a custom serializer was used; e.g., a different serializer than the one returned by TypeInformation#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 operator UnionState from a Savepoint.
        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 operator UnionState from a Savepoint when a custom serializer was used; e.g., a different serializer than the one returned by TypeInformation#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 operator BroadcastState from a Savepoint.
        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 operator BroadcastState from a Savepoint when a custom serializer was used; e.g., a different serializer than the one returned by TypeInformation#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 a Savepoint.
        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 - The KeyedStateReaderFunction 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 a Savepoint.
        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 - The KeyedStateReaderFunction 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 WindowWindowSavepointReader<W> window​(TypeSerializer<W> windowSerializer)
        Read window state from an operator in a Savepoint. This method supports reading from any type of window.
        Parameters:
        windowSerializer - The serializer used for the window type.
        Returns:
        A WindowSavepointReader.