State Schema Evolution

State Schema Evolution #

Apache Flink streaming applications are typically designed to run indefinitely or for long periods of time. As with all long-running services, the applications need to be updated to adapt to changing requirements. This goes the same for data schemas that the applications work against; they evolve along with the application.

This page provides an overview of how you can evolve your state type’s data schema. The current restrictions varies across different types and state structures (ValueState, ListState, etc.).

Note that the information on this page is relevant only if you are using state serializers that are generated by Flink’s own type serialization framework. That is, when declaring your state, the provided state descriptor is not configured to use a specific TypeSerializer or TypeInformation, in which case Flink infers information about the state type:

ListStateDescriptor<MyPojoType> descriptor =
    new ListStateDescriptor<>(
        "state-name",
        MyPojoType.class);

checkpointedState = getRuntimeContext().getListState(descriptor);

Under the hood, whether or not the schema of state can be evolved depends on the serializer used to read / write persisted state bytes. Simply put, a registered state’s schema can only be evolved if its serializer properly supports it. This is handled transparently by serializers generated by Flink’s type serialization framework (current scope of support is listed below.

If you intend to implement a custom TypeSerializer for your state type and would like to learn how to implement the serializer to support state schema evolution, please refer to Custom State Serialization. The documentation there also covers necessary internal details about the interplay between state serializers and Flink’s state backends to support state schema evolution.

Evolving state schema #

To evolve the schema of a given state type, you would take the following steps:

  1. Take a savepoint of your Flink streaming job.
  2. Update state types in your application (e.g., modifying your Avro type schema).
  3. Restore the job from the savepoint. When accessing state for the first time, Flink will assess whether or not the schema had been changed for the state, and migrate state schema if necessary.

The process of migrating state to adapt to changed schemas happens automatically, and independently for each state. This process is performed internally by Flink by first checking if the new serializer for the state has different serialization schema than the previous serializer; if so, the previous serializer is used to read the state to objects, and written back to bytes again with the new serializer.

Further details about the migration process is out of the scope of this documentation; please refer to here.

Supported data types for schema evolution #

Currently, schema evolution is supported only for POJO and Avro types. Therefore, if you care about schema evolution for state, it is currently recommended to always use either Pojo or Avro for state data types.

There are plans to extend the support for more composite types; for more details, please refer to FLINK-10896.

POJO types #

Flink supports evolving schema of POJO types, based on the following set of rules:

  1. Fields can be removed. Once removed, the previous value for the removed field will be dropped in future checkpoints and savepoints.
  2. New fields can be added. The new field will be initialized to the default value for its type, as defined by Java.
  3. Declared fields types cannot change.
  4. Class name of the POJO type cannot change, including the namespace of the class.

Note that the schema of POJO type state can only be evolved when restoring from a previous savepoint with Flink versions newer than 1.8.0. When restoring with Flink versions older than 1.8.0, the schema cannot be changed.

Avro types #

Flink fully supports evolving schema of Avro type state, as long as the schema change is considered compatible by Avro’s rules for schema resolution.

One limitation is that Avro generated classes used as the state type cannot be relocated or have different namespaces when the job is restored.

Schema Migration Limitations #

Flink’s schema migration has some limitations that are required to ensure correctness. For users that need to work around these limitations, and understand them to be safe in their specific use-case, consider using a custom serializer or the state processor api.

Schema evolution of keys is not supported. #

The structure of a key cannot be migrated as this may lead to non-deterministic behavior. For example, if a POJO is used as a key and one field is dropped then there may suddenly be multiple separate keys that are now identical. Flink has no way to merge the corresponding values.

Additionally, the RocksDB state backend relies on binary object identity, rather than the hashCode method. Any change to the keys’ object structure can lead to non-deterministic behavior.

Kryo cannot be used for schema evolution. #

When Kryo is used, there is no possibility for the framework to verify if any incompatible changes have been made.

This means that if a data-structure containing a given type is serialized via Kryo, then that contained type can not undergo schema evolution.

For example, if a POJO contains a List<SomeOtherPojo>, then the List and its contents are serialized via Kryo and schema evolution is not supported for SomeOtherPojo.

Back to top