Class TypeSerializer<T>
- java.lang.Object
-
- org.apache.flink.api.common.typeutils.TypeSerializer<T>
-
- Type Parameters:
T
- The data type that the serializer serializes.
- All Implemented Interfaces:
Serializable
- Direct Known Subclasses:
ArrayDataSerializer
,ArrayListSerializer
,AvroSerializer
,CoGroupedStreams.UnionSerializer
,CompositeSerializer
,CopyableValueSerializer
,DecimalDataSerializer
,DecimalDataSerializer
,EitherSerializer
,EnumSerializer
,ExternalSerializer
,GenericArraySerializer
,InternalTimersSnapshotReaderWriters.LegacyTimerSerializer
,IntervalJoinOperator.BufferEntrySerializer
,KryoSerializer
,LinkedListSerializer
,ListSerializer
,ListViewSerializer
,Lockable.LockableTypeSerializer
,MapDataSerializer
,MapSerializer
,MapViewSerializer
,NullableSerializer
,NullAwareMapSerializer
,PagedTypeSerializer
,PojoSerializer
,RawValueDataSerializer
,RowSerializer
,SimpleVersionedSerializerTypeSerializerProxy
,SingleThreadAccessCheckingTypeSerializer
,SortedMapSerializer
,StatefulComplexPayloadSerializer
,StreamElementSerializer
,TimerSerializer
,TimestampDataSerializer
,TtlAwareSerializer
,TupleSerializerBase
,TwoPhaseCommitSinkFunction.StateSerializer
,TypeSerializerSingleton
,UnloadableDummyTypeSerializer
,ValueSerializer
,WritableSerializer
@PublicEvolving public abstract class TypeSerializer<T> extends Object implements Serializable
This interface describes the methods that are required for a data type to be handled by the Flink runtime. Specifically, this interface contains the serialization and copying methods.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:
- Change the type serializer's config snapshot to implement
TypeSerializerSnapshot
, rather than extendingTypeSerializerConfigSnapshot
(as previously). - If the above step was completed, then the upgrade is done. Otherwise, if changing to
implement
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 (extendingTypeSerializerConfigSnapshot
) under the same name and give the updated serializer snapshot class (the one extendingTypeSerializerSnapshot
) a new name. - Implement the
TypeSerializerSnapshot.resolveSchemaCompatibility(TypeSerializerSnapshot)
method to perform the compatibility check.
- See Also:
- Serialized Form
-
-
Constructor Summary
Constructors Constructor Description TypeSerializer()
-
Method Summary
All Methods Instance Methods Abstract Methods Modifier and Type Method 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.
-
-
-
Method Detail
-
isImmutableType
public abstract boolean isImmutableType()
Gets whether the type is an immutable type.- Returns:
- True, if the type is immutable.
-
duplicate
public abstract TypeSerializer<T> duplicate()
Creates a deep copy of this serializer if it is necessary, i.e. if it is stateful. This can return itself if the serializer is not stateful.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.
-
createInstance
public abstract T createInstance()
Creates a new instance of the data type.- Returns:
- A new instance of the data type.
-
copy
public abstract T copy(T from)
Creates a deep copy of the given element in a new element.- Parameters:
from
- The element reuse be copied.- Returns:
- A deep copy of the element.
-
copy
public abstract T copy(T from, T reuse)
Creates a copy from the given element. The method makes an attempt to store the copy in the given reuse element, if the type is mutable. This is, however, not guaranteed.- Parameters:
from
- The element to be copied.reuse
- The element to be reused. May or may not be used.- Returns:
- A deep copy of the element.
-
getLength
public abstract int getLength()
Gets the length of the data type, if it is a fix length data type.- Returns:
- The length of the data type, or
-1
for variable length data types.
-
serialize
public abstract void serialize(T record, DataOutputView target) throws IOException
Serializes the given record to the given target output view.- Parameters:
record
- The record to serialize.target
- The output view to write the serialized data to.- Throws:
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.
-
deserialize
public abstract T deserialize(DataInputView source) throws IOException
De-serializes a record from the given source input view.- Parameters:
source
- The input view from which to read the data.- Returns:
- The deserialized element.
- Throws:
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.
-
deserialize
public abstract T deserialize(T reuse, DataInputView source) throws IOException
De-serializes a record from the given source input view into the given reuse record instance if mutable.- Parameters:
reuse
- The record instance into which to de-serialize the data.source
- The input view from which to read the data.- Returns:
- The deserialized element.
- Throws:
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.
-
copy
public abstract void copy(DataInputView source, DataOutputView target) throws IOException
Copies exactly one record from the source input view to the target output view. Whether this operation works on binary data or partially de-serializes the record to determine its length (such as for records of variable length) is up to the implementer. Binary copies are typically faster. A copy of a record containing two integer numbers (8 bytes total) is most efficiently implemented astarget.write(source, 8);
.- Parameters:
source
- The input view from which to read the record.target
- The target output view to which to write the record.- Throws:
IOException
- Thrown if any of the two views raises an exception.
-
snapshotConfiguration
public abstract TypeSerializerSnapshot<T> snapshotConfiguration()
Snapshots the configuration of this TypeSerializer. This method is only relevant if the serializer is used to state stored in checkpoints/savepoints.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.
- Returns:
- snapshot of the serializer's current configuration (cannot be
null
). - See Also:
TypeSerializerSnapshot.resolveSchemaCompatibility(TypeSerializerSnapshot)
-
-