Class BinaryRowDataSerializer
- java.lang.Object
-
- org.apache.flink.api.common.typeutils.TypeSerializer<T>
-
- org.apache.flink.table.runtime.typeutils.PagedTypeSerializer<T>
-
- org.apache.flink.table.runtime.typeutils.AbstractRowDataSerializer<BinaryRowData>
-
- org.apache.flink.table.runtime.typeutils.BinaryRowDataSerializer
-
- All Implemented Interfaces:
Serializable
@Internal public class BinaryRowDataSerializer extends AbstractRowDataSerializer<BinaryRowData>
Serializer forBinaryRowData
.- See Also:
- Serialized Form
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static class
BinaryRowDataSerializer.BinaryRowDataSerializerSnapshot
-
Field Summary
Fields Modifier and Type Field Description static int
LENGTH_SIZE_IN_BYTES
-
Constructor Summary
Constructors Constructor Description BinaryRowDataSerializer(int numFields)
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description void
checkSkipReadForFixLengthPart(AbstractPagedInputView source)
We need skip bytes to read when the remain bytes of current segment is not enough to write binary row fixed part.void
copy(DataInputView source, DataOutputView target)
Copies exactly one record from the source input view to the target output view.BinaryRowData
copy(BinaryRowData from)
Creates a deep copy of the given element in a new element.BinaryRowData
copy(BinaryRowData from, BinaryRowData reuse)
Creates a copy from the given element.void
copyFromPagesToView(AbstractPagedInputView source, DataOutputView target)
Copy a binaryRow which stored in paged input view to output view.BinaryRowData
createInstance()
Creates a new instance of the data type.BinaryRowData
deserialize(DataInputView source)
De-serializes a record from the given source input view.BinaryRowData
deserialize(BinaryRowData reuse, DataInputView source)
De-serializes a record from the given source input view into the given reuse record instance if mutable.BinaryRowData
deserializeFromPages(AbstractPagedInputView headerLessView)
De-serializes a record from the given source paged input view.BinaryRowData
deserializeFromPages(BinaryRowData reuse, AbstractPagedInputView headerLessView)
Reuse version ofPagedTypeSerializer.deserializeFromPages(AbstractPagedInputView)
.TypeSerializer<BinaryRowData>
duplicate()
Creates a deep copy of this serializer if it is necessary, i.e. if it is stateful.boolean
equals(Object obj)
int
getArity()
Get the number of fields.int
getFixedLengthPartSize()
int
getLength()
Gets the length of the data type, if it is a fix length data type.int
getSerializedRowFixedPartLength()
Return fixed part length to serialize one row.int
hashCode()
boolean
isImmutableType()
Gets whether the type is an immutable type.BinaryRowData
mapFromPages(BinaryRowData reuse, AbstractPagedInputView headerLessView)
Map a reused record from the given source paged input view.void
pointTo(int length, BinaryRowData reuse, AbstractPagedInputView headerLessView)
Point row to memory segments with offset(in the AbstractPagedInputView) and length.void
serialize(BinaryRowData record, DataOutputView target)
Serializes the given record to the given target output view.int
serializeToPages(BinaryRowData record, AbstractPagedOutputView headerLessView)
Serializes the given record to the given target paged output view.static void
serializeWithoutLengthSlow(BinaryRowData record, MemorySegmentWritable out)
void
skipRecordFromPages(AbstractPagedInputView headerLessView)
Skip over bytes of one record from the paged input view, discarding the skipped bytes.TypeSerializerSnapshot<BinaryRowData>
snapshotConfiguration()
Snapshots the configuration of this TypeSerializer.BinaryRowData
toBinaryRow(BinaryRowData rowData)
Convert aRowData
to aBinaryRowData
.
-
-
-
Field Detail
-
LENGTH_SIZE_IN_BYTES
public static final int LENGTH_SIZE_IN_BYTES
- See Also:
- Constant Field Values
-
-
Method Detail
-
isImmutableType
public boolean isImmutableType()
Description copied from class:TypeSerializer
Gets whether the type is an immutable type.- Specified by:
isImmutableType
in classTypeSerializer<BinaryRowData>
- Returns:
- True, if the type is immutable.
-
duplicate
public TypeSerializer<BinaryRowData> duplicate()
Description copied from class:TypeSerializer
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.
- Specified by:
duplicate
in classTypeSerializer<BinaryRowData>
-
createInstance
public BinaryRowData createInstance()
Description copied from class:TypeSerializer
Creates a new instance of the data type.- Specified by:
createInstance
in classTypeSerializer<BinaryRowData>
- Returns:
- A new instance of the data type.
-
copy
public BinaryRowData copy(BinaryRowData from)
Description copied from class:TypeSerializer
Creates a deep copy of the given element in a new element.- Specified by:
copy
in classTypeSerializer<BinaryRowData>
- Parameters:
from
- The element reuse be copied.- Returns:
- A deep copy of the element.
-
copy
public BinaryRowData copy(BinaryRowData from, BinaryRowData reuse)
Description copied from class:TypeSerializer
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.- Specified by:
copy
in classTypeSerializer<BinaryRowData>
- 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 int getLength()
Description copied from class:TypeSerializer
Gets the length of the data type, if it is a fix length data type.- Specified by:
getLength
in classTypeSerializer<BinaryRowData>
- Returns:
- The length of the data type, or
-1
for variable length data types.
-
serialize
public void serialize(BinaryRowData record, DataOutputView target) throws IOException
Description copied from class:TypeSerializer
Serializes the given record to the given target output view.- Specified by:
serialize
in classTypeSerializer<BinaryRowData>
- 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 BinaryRowData deserialize(DataInputView source) throws IOException
Description copied from class:TypeSerializer
De-serializes a record from the given source input view.- Specified by:
deserialize
in classTypeSerializer<BinaryRowData>
- 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 BinaryRowData deserialize(BinaryRowData reuse, DataInputView source) throws IOException
Description copied from class:TypeSerializer
De-serializes a record from the given source input view into the given reuse record instance if mutable.- Specified by:
deserialize
in classTypeSerializer<BinaryRowData>
- 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.
-
getArity
public int getArity()
Description copied from class:AbstractRowDataSerializer
Get the number of fields.- Specified by:
getArity
in classAbstractRowDataSerializer<BinaryRowData>
-
toBinaryRow
public BinaryRowData toBinaryRow(BinaryRowData rowData) throws IOException
Description copied from class:AbstractRowDataSerializer
Convert aRowData
to aBinaryRowData
.- Specified by:
toBinaryRow
in classAbstractRowDataSerializer<BinaryRowData>
- Throws:
IOException
-
serializeToPages
public int serializeToPages(BinaryRowData record, AbstractPagedOutputView headerLessView) throws IOException
Description copied from class:PagedTypeSerializer
Serializes the given record to the given target paged output view. Some implementations may skip some bytes if current page does not have enough space left, .e.gBinaryRowData
.- Specified by:
serializeToPages
in classPagedTypeSerializer<BinaryRowData>
- Parameters:
record
- The record to serialize.headerLessView
- The output view to write the serialized data to.- Returns:
- Returns the skipped number of bytes.
- 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.
-
serializeWithoutLengthSlow
public static void serializeWithoutLengthSlow(BinaryRowData record, MemorySegmentWritable out) throws IOException
- Throws:
IOException
-
deserializeFromPages
public BinaryRowData deserializeFromPages(AbstractPagedInputView headerLessView) throws IOException
Description copied from class:PagedTypeSerializer
De-serializes a record from the given source paged input view. For consistency with serialize format, some implementations may need to skip some bytes of source before de-serializing, .e.gBinaryRowData
. Typically, the content read from source should be copied out when de-serializing, and we are not expecting the underlying data from source is reused. If you have such requirement, see#mapFromPages(T, AbstractPagedInputView)
.- Specified by:
deserializeFromPages
in classPagedTypeSerializer<BinaryRowData>
- Parameters:
headerLessView
- The input view from which to read the data.- Returns:
- The de-serialized 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.
-
deserializeFromPages
public BinaryRowData deserializeFromPages(BinaryRowData reuse, AbstractPagedInputView headerLessView) throws IOException
Description copied from class:PagedTypeSerializer
Reuse version ofPagedTypeSerializer.deserializeFromPages(AbstractPagedInputView)
.- Specified by:
deserializeFromPages
in classPagedTypeSerializer<BinaryRowData>
- Throws:
IOException
-
mapFromPages
public BinaryRowData mapFromPages(BinaryRowData reuse, AbstractPagedInputView headerLessView) throws IOException
Description copied from class:PagedTypeSerializer
Map a reused record from the given source paged input view. This method provides a possibility to achieve zero copy when de-serializing. You can either choose copy or not copy the content read from source, but we encourage to make it zero copy.If you choose the zero copy way, you have to deal with the lifecycle of the pages properly.
- Specified by:
mapFromPages
in classPagedTypeSerializer<BinaryRowData>
- Parameters:
reuse
- the reused record to be mappedheaderLessView
- The input view from which to read the data.- Returns:
- The mapped record.
- 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.
-
skipRecordFromPages
public void skipRecordFromPages(AbstractPagedInputView headerLessView) throws IOException
Description copied from class:PagedTypeSerializer
Skip over bytes of one record from the paged input view, discarding the skipped bytes.- Specified by:
skipRecordFromPages
in classPagedTypeSerializer<BinaryRowData>
- Throws:
IOException
-
copyFromPagesToView
public void copyFromPagesToView(AbstractPagedInputView source, DataOutputView target) throws IOException
Copy a binaryRow which stored in paged input view to output view.- Parameters:
source
- source paged input view where the binary row storedtarget
- the target output view.- Throws:
IOException
-
pointTo
public void pointTo(int length, BinaryRowData reuse, AbstractPagedInputView headerLessView) throws IOException
Point row to memory segments with offset(in the AbstractPagedInputView) and length.- Parameters:
length
- row length.reuse
- reuse BinaryRowData object.headerLessView
- source memory segments container.- Throws:
IOException
-
checkSkipReadForFixLengthPart
public void checkSkipReadForFixLengthPart(AbstractPagedInputView source) throws IOException
We need skip bytes to read when the remain bytes of current segment is not enough to write binary row fixed part. SeeBinaryRowData
.- Throws:
IOException
-
getSerializedRowFixedPartLength
public int getSerializedRowFixedPartLength()
Return fixed part length to serialize one row.
-
getFixedLengthPartSize
public int getFixedLengthPartSize()
-
copy
public void copy(DataInputView source, DataOutputView target) throws IOException
Description copied from class:TypeSerializer
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);
.- Specified by:
copy
in classTypeSerializer<BinaryRowData>
- 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.
-
equals
public boolean equals(Object obj)
- Specified by:
equals
in classTypeSerializer<BinaryRowData>
-
hashCode
public int hashCode()
- Specified by:
hashCode
in classTypeSerializer<BinaryRowData>
-
snapshotConfiguration
public TypeSerializerSnapshot<BinaryRowData> snapshotConfiguration()
Description copied from class:TypeSerializer
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.
- Specified by:
snapshotConfiguration
in classTypeSerializer<BinaryRowData>
- Returns:
- snapshot of the serializer's current configuration (cannot be
null
). - See Also:
TypeSerializerSnapshot.resolveSchemaCompatibility(TypeSerializerSnapshot)
-
-