T
- The data type that the serializer serializes.@PublicEvolving public abstract class TypeSerializer<T> extends Object implements Serializable
The methods in this class are not necessarily thread safe. To avoid unpredictable side
effects, it is recommended to call duplicate()
method and use one serializer instance per
thread.
Upgrading TypeSerializers to the new TypeSerializerSnapshot model
This section is relevant if you implemented a TypeSerializer in Flink versions up to 1.6 and want to adapt that implementation to the new interfaces that support proper state schema evolution, while maintaining backwards compatibility. Please follow these steps:
TypeSerializerSnapshot
,
rather than extending TypeSerializerConfigSnapshot
(as previously).
TypeSerializerSnapshot
directly in-place as the same class isn't possible
(perhaps because the new snapshot is intended to have completely different written contents
or intended to have a different class name), retain the old serializer snapshot class
(extending TypeSerializerConfigSnapshot
) under the same name and give the updated
serializer snapshot class (the one extending TypeSerializerSnapshot
) a new name.
TypeSerializerSnapshot.resolveSchemaCompatibility(TypeSerializerSnapshot)
method to
perform the compatibility check.
Constructor and Description |
---|
TypeSerializer() |
Modifier and Type | Method and Description |
---|---|
abstract void |
copy(DataInputView source,
DataOutputView target)
Copies exactly one record from the source input view to the target output view.
|
abstract T |
copy(T from)
Creates a deep copy of the given element in a new element.
|
abstract T |
copy(T from,
T reuse)
Creates a copy from the given element.
|
abstract T |
createInstance()
Creates a new instance of the data type.
|
abstract T |
deserialize(DataInputView source)
De-serializes a record from the given source input view.
|
abstract T |
deserialize(T reuse,
DataInputView source)
De-serializes a record from the given source input view into the given reuse record instance
if mutable.
|
abstract TypeSerializer<T> |
duplicate()
Creates a deep copy of this serializer if it is necessary, i.e. if it is stateful.
|
abstract boolean |
equals(Object obj) |
abstract int |
getLength()
Gets the length of the data type, if it is a fix length data type.
|
abstract int |
hashCode() |
abstract boolean |
isImmutableType()
Gets whether the type is an immutable type.
|
abstract void |
serialize(T record,
DataOutputView target)
Serializes the given record to the given target output view.
|
abstract TypeSerializerSnapshot<T> |
snapshotConfiguration()
Snapshots the configuration of this TypeSerializer.
|
public abstract boolean isImmutableType()
public abstract TypeSerializer<T> duplicate()
We need this because Serializers might be used in several threads. Stateless serializers are inherently thread-safe while stateful serializers might not be thread-safe.
public abstract T createInstance()
public abstract T copy(T from)
from
- The element reuse be copied.public abstract T copy(T from, T reuse)
from
- The element to be copied.reuse
- The element to be reused. May or may not be used.public abstract int getLength()
-1
for variable length data types.public abstract void serialize(T record, DataOutputView target) throws IOException
record
- The record to serialize.target
- The output view to write the serialized data to.IOException
- Thrown, if the serialization encountered an I/O related error. Typically
raised by the output view, which may have an underlying I/O channel to which it
delegates.public abstract T deserialize(DataInputView source) throws IOException
source
- The input view from which to read the data.IOException
- Thrown, if the de-serialization encountered an I/O related error.
Typically raised by the input view, which may have an underlying I/O channel from which
it reads.public abstract T deserialize(T reuse, DataInputView source) throws IOException
reuse
- The record instance into which to de-serialize the data.source
- The input view from which to read the data.IOException
- Thrown, if the de-serialization encountered an I/O related error.
Typically raised by the input view, which may have an underlying I/O channel from which
it reads.public abstract void copy(DataInputView source, DataOutputView target) throws IOException
target.write(source, 8);
.source
- The input view from which to read the record.target
- The target output view to which to write the record.IOException
- Thrown if any of the two views raises an exception.public abstract TypeSerializerSnapshot<T> snapshotConfiguration()
The snapshot of the TypeSerializer is supposed to contain all information that affects the serialization format of the serializer. The snapshot serves two purposes: First, to reproduce the serializer when the checkpoint/savepoint is restored, and second, to check whether the serialization format is compatible with the serializer used in the restored program.
IMPORTANT: TypeSerializerSnapshots changed after Flink 1.6. Serializers implemented against Flink versions up to 1.6 should still work, but adjust to new model to enable state evolution and be future-proof. See the class-level comments, section "Upgrading TypeSerializers to the new TypeSerializerSnapshot model" for details.
null
).TypeSerializerSnapshot.resolveSchemaCompatibility(TypeSerializerSnapshot)
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.