Interface DeserializationSchema<T>
-
- Type Parameters:
T
- The type created by the deserialization schema.
- All Superinterfaces:
ResultTypeQueryable<T>
,Serializable
- All Known Implementing Classes:
AbstractDeserializationSchema
,AbstractJsonDeserializationSchema
,AvroDeserializationSchema
,AvroRowDataDeserializationSchema
,AvroRowDeserializationSchema
,CanalJsonDeserializationSchema
,ChangelogCsvDeserializer
,ConfluentRegistryAvroDeserializationSchema
,CsvRowDataDeserializationSchema
,CsvRowDeserializationSchema
,DebeziumAvroDeserializationSchema
,DebeziumJsonDeserializationSchema
,EventDeSerializationSchema
,JsonDeserializationSchema
,JsonParserRowDataDeserializationSchema
,JsonRowDataDeserializationSchema
,JsonRowDeserializationSchema
,MaxwellJsonDeserializationSchema
,OggJsonDeserializationSchema
,PbRowDataDeserializationSchema
,RawFormatDeserializationSchema
,RegistryAvroDeserializationSchema
,SimpleStringSchema
,TypeInformationSerializationSchema
@Public public interface DeserializationSchema<T> extends Serializable, ResultTypeQueryable<T>
The deserialization schema describes how to turn the byte messages delivered by certain data sources (for example Apache Kafka) into data types (Java/Scala objects) that are processed by Flink.In addition, the DeserializationSchema describes the produced type (
ResultTypeQueryable.getProducedType()
), which lets Flink create internal serializers and structures to handle the type.Note: In most cases, one should start from
AbstractDeserializationSchema
, which takes care of producing the return type information automatically.A DeserializationSchema must be
Serializable
because its instances are often part of an operator or transformation function.
-
-
Nested Class Summary
Nested Classes Modifier and Type Interface Description static interface
DeserializationSchema.InitializationContext
A contextual information provided foropen(InitializationContext)
method.
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description T
deserialize(byte[] message)
Deserializes the byte message.default void
deserialize(byte[] message, Collector<T> out)
Deserializes the byte message.boolean
isEndOfStream(T nextElement)
Method to decide whether the element signals the end of the stream.default void
open(DeserializationSchema.InitializationContext context)
Initialization method for the schema.-
Methods inherited from interface org.apache.flink.api.java.typeutils.ResultTypeQueryable
getProducedType
-
-
-
-
Method Detail
-
open
@PublicEvolving default void open(DeserializationSchema.InitializationContext context) throws Exception
Initialization method for the schema. It is called before the actual working methodsdeserialize(byte[])
and thus suitable for one time setup work.The provided
DeserializationSchema.InitializationContext
can be used to access additional features such as e.g. registering user metrics.- Parameters:
context
- Contextual information that can be used during initialization.- Throws:
Exception
-
deserialize
T deserialize(byte[] message) throws IOException
Deserializes the byte message.- Parameters:
message
- The message, as a byte array.- Returns:
- The deserialized message as an object (null if the message cannot be deserialized).
- Throws:
IOException
-
deserialize
@PublicEvolving default void deserialize(byte[] message, Collector<T> out) throws IOException
Deserializes the byte message.Can output multiple records through the
Collector
. Note that number and size of the produced records should be relatively small. Depending on the source implementation records can be buffered in memory or collecting records might delay emitting checkpoint barrier.- Parameters:
message
- The message, as a byte array.out
- The collector to put the resulting messages.- Throws:
IOException
-
isEndOfStream
boolean isEndOfStream(T nextElement)
Method to decide whether the element signals the end of the stream. If true is returned the element won't be emitted.- Parameters:
nextElement
- The element to test for the end-of-stream signal.- Returns:
- True, if the element signals end of stream, false otherwise.
-
-