Class RecordWriter<T extends IOReadableWritable>
- java.lang.Object
-
- org.apache.flink.runtime.io.network.api.writer.RecordWriter<T>
-
- Type Parameters:
T
- the type of the record that can be emitted with this record writer
- All Implemented Interfaces:
AvailabilityProvider
- Direct Known Subclasses:
BroadcastRecordWriter
,ChannelSelectorRecordWriter
public abstract class RecordWriter<T extends IOReadableWritable> extends Object implements AvailabilityProvider
An abstract record-oriented runtime result writer.The RecordWriter wraps the runtime's
ResultPartitionWriter
and takes care of subpartition selection and serializing records into bytes.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AvailabilityProvider.AvailabilityHelper
-
-
Field Summary
Fields Modifier and Type Field Description static String
DEFAULT_OUTPUT_FLUSH_THREAD_NAME
Default name for the output flush thread, if no name with a task reference is given.protected boolean
flushAlways
protected int
numberOfSubpartitions
protected Random
rng
protected DataOutputSerializer
serializer
protected ResultPartitionWriter
targetPartition
-
Fields inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
AVAILABLE
-
-
Method Summary
All Methods Static Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
abortCheckpoint(long checkpointId, CheckpointException cause)
void
alignedBarrierTimeout(long checkpointId)
abstract void
broadcastEmit(T record)
This is used to broadcast streaming Watermarks in-band with records.void
broadcastEvent(AbstractEvent event)
void
broadcastEvent(AbstractEvent event, boolean isPriorityEvent)
protected void
checkErroneous()
void
close()
Closes the writer.protected void
emit(ByteBuffer record, int targetSubpartition)
abstract void
emit(T record)
This is used to send regular records.void
emit(T record, int targetSubpartition)
void
flushAll()
CompletableFuture<?>
getAvailableFuture()
int
getNumberOfSubpartitions()
boolean
isSubpartitionDerivable()
Whether the subpartition where an element comes from can be derived from the existing information.void
randomEmit(T record)
This is used to send LatencyMarks to a random target subpartition.static ByteBuffer
serializeRecord(DataOutputSerializer serializer, IOReadableWritable record)
void
setMaxOverdraftBuffersPerGate(int maxOverdraftBuffersPerGate)
Sets the max overdraft buffer size of per gate.void
setMetricGroup(TaskIOMetricGroup metrics)
Sets the metric group for this RecordWriter.-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface org.apache.flink.runtime.io.AvailabilityProvider
isApproximatelyAvailable, isAvailable
-
-
-
-
Field Detail
-
DEFAULT_OUTPUT_FLUSH_THREAD_NAME
@VisibleForTesting public static final String DEFAULT_OUTPUT_FLUSH_THREAD_NAME
Default name for the output flush thread, if no name with a task reference is given.- See Also:
- Constant Field Values
-
targetPartition
protected final ResultPartitionWriter targetPartition
-
numberOfSubpartitions
protected final int numberOfSubpartitions
-
serializer
protected final DataOutputSerializer serializer
-
rng
protected final Random rng
-
flushAlways
protected final boolean flushAlways
-
-
Method Detail
-
emit
public void emit(T record, int targetSubpartition) throws IOException
- Throws:
IOException
-
emit
protected void emit(ByteBuffer record, int targetSubpartition) throws IOException
- Throws:
IOException
-
broadcastEvent
public void broadcastEvent(AbstractEvent event) throws IOException
- Throws:
IOException
-
broadcastEvent
public void broadcastEvent(AbstractEvent event, boolean isPriorityEvent) throws IOException
- Throws:
IOException
-
alignedBarrierTimeout
public void alignedBarrierTimeout(long checkpointId) throws IOException
- Throws:
IOException
-
abortCheckpoint
public void abortCheckpoint(long checkpointId, CheckpointException cause)
-
serializeRecord
@VisibleForTesting public static ByteBuffer serializeRecord(DataOutputSerializer serializer, IOReadableWritable record) throws IOException
- Throws:
IOException
-
flushAll
public void flushAll()
-
setMetricGroup
public void setMetricGroup(TaskIOMetricGroup metrics)
Sets the metric group for this RecordWriter.
-
getNumberOfSubpartitions
public int getNumberOfSubpartitions()
-
isSubpartitionDerivable
public boolean isSubpartitionDerivable()
Whether the subpartition where an element comes from can be derived from the existing information. If false, the caller of this writer should attach the subpartition information onto an element before writing it to a subpartition, if the element needs this information afterward.
-
getAvailableFuture
public CompletableFuture<?> getAvailableFuture()
- Specified by:
getAvailableFuture
in interfaceAvailabilityProvider
- Returns:
- a future that is completed if the respective provider is available.
-
emit
public abstract void emit(T record) throws IOException
This is used to send regular records.- Throws:
IOException
-
randomEmit
public void randomEmit(T record) throws IOException
This is used to send LatencyMarks to a random target subpartition.- Throws:
IOException
-
broadcastEmit
public abstract void broadcastEmit(T record) throws IOException
This is used to broadcast streaming Watermarks in-band with records.- Throws:
IOException
-
close
public void close()
Closes the writer. This stops the flushing thread (if there is one).
-
checkErroneous
protected void checkErroneous() throws IOException
- Throws:
IOException
-
setMaxOverdraftBuffersPerGate
public void setMaxOverdraftBuffersPerGate(int maxOverdraftBuffersPerGate)
Sets the max overdraft buffer size of per gate.
-
-