Class FileRegionBuffer
- java.lang.Object
-
- org.apache.flink.shaded.netty4.io.netty.util.AbstractReferenceCounted
-
- org.apache.flink.shaded.netty4.io.netty.channel.DefaultFileRegion
-
- org.apache.flink.runtime.io.network.buffer.FileRegionBuffer
-
- All Implemented Interfaces:
Buffer
,org.apache.flink.shaded.netty4.io.netty.channel.FileRegion
,org.apache.flink.shaded.netty4.io.netty.util.ReferenceCounted
public class FileRegionBuffer extends org.apache.flink.shaded.netty4.io.netty.channel.DefaultFileRegion implements Buffer
This class represents a chunk of data in a file channel. Its purpose is to be passed to the netty code and to be written to the socket via the zero-copy direct transfer capabilities ofFileChannel.transferTo(long, long, WritableByteChannel)
.This class implements
Buffer
mainly for compatible with existing usages. It can be thought of as a "lazy buffer" that does not hold the data directly, although the data can be fetches as a read-onlyByteBuffer
when needed, for example in local input channels. SeereadInto(MemorySegment)
andgetNioBufferReadable()
. Because this buffer is read-only, the modification methods (and methods that give a writable buffer) throwUnsupportedOperationException
.This class extends from Netty's
DefaultFileRegion
, similar as theNetworkBuffer
extends from Netty'sByteBuf
. That way we can pass both of them to Netty in the same way, and Netty will internally treat them appropriately.
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from interface org.apache.flink.runtime.io.network.buffer.Buffer
Buffer.DataType
-
-
Constructor Summary
Constructors Constructor Description FileRegionBuffer(FileChannel fileChannel, long fileChannelPosition, int bufferSize, Buffer.DataType dataType, boolean isCompressed)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description org.apache.flink.shaded.netty4.io.netty.buffer.ByteBuf
asByteBuf()
void
deallocate()
Buffer.DataType
getDataType()
Gets the type of data this buffer represents.int
getMaxCapacity()
Returns the maximum size of the buffer, i.e. the capacity of the underlyingMemorySegment
.MemorySegment
getMemorySegment()
Returns the underlying memory segment.int
getMemorySegmentOffset()
This method will be removed in the future.ByteBuffer
getNioBuffer(int index, int length)
Gets a newByteBuffer
instance wrapping this buffer's bytes.ByteBuffer
getNioBufferReadable()
This method is only called by tests and by event-deserialization, like checkpoint barriers.int
getReaderIndex()
Returns the reader index of this buffer.BufferRecycler
getRecycler()
Gets the buffer's recycler.int
getSize()
Returns the size of the written data, i.e. the writer index, of this buffer.boolean
isBuffer()
Returns whether this buffer represents a buffer or an event.boolean
isCompressed()
boolean
isRecycled()
Returns whether this buffer has been recycled or not.int
readableBytes()
Returns the number of readable bytes (same asBuffer.getSize()
-Buffer.getReaderIndex()
).Buffer
readInto(MemorySegment segment)
ReadOnlySlicedNetworkBuffer
readOnlySlice()
Returns a read-only slice of this buffer's readable bytes, i.e. betweenBuffer.getReaderIndex()
andBuffer.getSize()
.ReadOnlySlicedNetworkBuffer
readOnlySlice(int index, int length)
Returns a read-only slice of this buffer.void
recycleBuffer()
Releases this buffer once, i.e. reduces the reference count and recycles the buffer if the reference count reaches 0.FileRegionBuffer
retainBuffer()
Retains this buffer for further use, increasing the reference counter by 1.void
setAllocator(org.apache.flink.shaded.netty4.io.netty.buffer.ByteBufAllocator allocator)
Sets the buffer allocator for use in netty.void
setCompressed(boolean isCompressed)
Tags the buffer as compressed or uncompressed.void
setDataType(Buffer.DataType dataType)
Sets the type of data this buffer represents.void
setReaderIndex(int readerIndex)
Sets the reader index of this buffer.void
setRecycler(BufferRecycler bufferRecycler)
Sets the buffer's recycler.void
setSize(int writerIndex)
Sets the size of the written data, i.e. the writer index, of this buffer.-
Methods inherited from class org.apache.flink.shaded.netty4.io.netty.channel.DefaultFileRegion
count, isOpen, open, position, retain, retain, touch, touch, transfered, transferred, transferTo
-
Methods inherited from class org.apache.flink.shaded.netty4.io.netty.util.AbstractReferenceCounted
refCnt, release, release, setRefCnt
-
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.network.buffer.Buffer
refCnt, toDebugString
-
-
-
-
Constructor Detail
-
FileRegionBuffer
public FileRegionBuffer(FileChannel fileChannel, long fileChannelPosition, int bufferSize, Buffer.DataType dataType, boolean isCompressed)
-
-
Method Detail
-
isBuffer
public boolean isBuffer()
Description copied from interface:Buffer
Returns whether this buffer represents a buffer or an event.
-
getMemorySegment
public MemorySegment getMemorySegment()
Description copied from interface:Buffer
Returns the underlying memory segment. This method is dangerous since it ignores read only protections and omits slices. Use it only along theBuffer.getMemorySegmentOffset()
.This method will be removed in the future. For writing use
BufferBuilder
.- Specified by:
getMemorySegment
in interfaceBuffer
- Returns:
- the memory segment backing this buffer
-
getMemorySegmentOffset
public int getMemorySegmentOffset()
Description copied from interface:Buffer
This method will be removed in the future. For writing useBufferBuilder
.- Specified by:
getMemorySegmentOffset
in interfaceBuffer
- Returns:
- the offset where this (potential slice)
Buffer
's data start in the underlying memory segment.
-
readOnlySlice
public ReadOnlySlicedNetworkBuffer readOnlySlice()
Description copied from interface:Buffer
Returns a read-only slice of this buffer's readable bytes, i.e. betweenBuffer.getReaderIndex()
andBuffer.getSize()
.Reader and writer indices as well as markers are not shared. Reference counters are shared but the slice is not
retained
automatically.- Specified by:
readOnlySlice
in interfaceBuffer
- Returns:
- a read-only sliced buffer
-
readOnlySlice
public ReadOnlySlicedNetworkBuffer readOnlySlice(int index, int length)
Description copied from interface:Buffer
Returns a read-only slice of this buffer.Reader and writer indices as well as markers are not shared. Reference counters are shared but the slice is not
retained
automatically.- Specified by:
readOnlySlice
in interfaceBuffer
- Parameters:
index
- the index to start fromlength
- the length of the slice- Returns:
- a read-only sliced buffer
-
getMaxCapacity
public int getMaxCapacity()
Description copied from interface:Buffer
Returns the maximum size of the buffer, i.e. the capacity of the underlyingMemorySegment
.- Specified by:
getMaxCapacity
in interfaceBuffer
- Returns:
- size of the buffer
-
getReaderIndex
public int getReaderIndex()
Description copied from interface:Buffer
Returns the reader index of this buffer.This is where readable (unconsumed) bytes start in the backing memory segment.
- Specified by:
getReaderIndex
in interfaceBuffer
- Returns:
- reader index (from 0 (inclusive) to the size of the backing
MemorySegment
(inclusive))
-
setReaderIndex
public void setReaderIndex(int readerIndex) throws IndexOutOfBoundsException
Description copied from interface:Buffer
Sets the reader index of this buffer.- Specified by:
setReaderIndex
in interfaceBuffer
- Throws:
IndexOutOfBoundsException
- if the index is less than 0 or greater thanBuffer.getSize()
-
getNioBufferReadable
public ByteBuffer getNioBufferReadable()
This method is only called by tests and by event-deserialization, like checkpoint barriers. Because such events are not used for bounded intermediate results, this method currently executes only in tests.- Specified by:
getNioBufferReadable
in interfaceBuffer
- Returns:
- byte buffer sharing the contents of the underlying memory segment
-
getNioBuffer
public ByteBuffer getNioBuffer(int index, int length) throws IndexOutOfBoundsException
Description copied from interface:Buffer
Gets a newByteBuffer
instance wrapping this buffer's bytes.Please note that neither read nor write index are updated by the returned buffer.
- Specified by:
getNioBuffer
in interfaceBuffer
- Returns:
- byte buffer sharing the contents of the underlying memory segment
- Throws:
IndexOutOfBoundsException
- if the indexes are not without the buffer's bounds- See Also:
Buffer.getNioBufferReadable()
-
asByteBuf
public org.apache.flink.shaded.netty4.io.netty.buffer.ByteBuf asByteBuf()
-
setSize
public void setSize(int writerIndex)
Description copied from interface:Buffer
Sets the size of the written data, i.e. the writer index, of this buffer.
-
getSize
public int getSize()
Description copied from interface:Buffer
Returns the size of the written data, i.e. the writer index, of this buffer.This is where writable bytes start in the backing memory segment.
- Specified by:
getSize
in interfaceBuffer
- Returns:
- writer index (from 0 (inclusive) to the size of the backing
MemorySegment
(inclusive))
-
readableBytes
public int readableBytes()
Description copied from interface:Buffer
Returns the number of readable bytes (same asBuffer.getSize()
-Buffer.getReaderIndex()
).- Specified by:
readableBytes
in interfaceBuffer
-
setAllocator
public void setAllocator(org.apache.flink.shaded.netty4.io.netty.buffer.ByteBufAllocator allocator)
Description copied from interface:Buffer
Sets the buffer allocator for use in netty.- Specified by:
setAllocator
in interfaceBuffer
- Parameters:
allocator
- netty buffer allocator
-
getRecycler
public BufferRecycler getRecycler()
Description copied from interface:Buffer
Gets the buffer's recycler.- Specified by:
getRecycler
in interfaceBuffer
- Returns:
- buffer recycler
-
setRecycler
public void setRecycler(BufferRecycler bufferRecycler)
Description copied from interface:Buffer
Sets the buffer's recycler.Note that updating the recycler is an unsafe operation and this method cannot guarantee thread safety. It is important for the caller to fully understand the consequences of calling this method. Incorrectly updating the buffer recycler can result in a leak of the buffer due to using a wrong recycler to recycle buffer. Therefore, be careful when calling this method.
- Specified by:
setRecycler
in interfaceBuffer
- Parameters:
bufferRecycler
- the new buffer recycler
-
recycleBuffer
public void recycleBuffer()
Description copied from interface:Buffer
Releases this buffer once, i.e. reduces the reference count and recycles the buffer if the reference count reaches 0.- Specified by:
recycleBuffer
in interfaceBuffer
- See Also:
Buffer.retainBuffer()
-
isRecycled
public boolean isRecycled()
Description copied from interface:Buffer
Returns whether this buffer has been recycled or not.- Specified by:
isRecycled
in interfaceBuffer
- Returns:
- true if already recycled, false otherwise
-
retainBuffer
public FileRegionBuffer retainBuffer()
Description copied from interface:Buffer
Retains this buffer for further use, increasing the reference counter by 1.- Specified by:
retainBuffer
in interfaceBuffer
- Returns:
- this instance (for chained calls)
- See Also:
Buffer.recycleBuffer()
-
isCompressed
public boolean isCompressed()
- Specified by:
isCompressed
in interfaceBuffer
- Returns:
- whether the buffer is compressed or not.
-
setCompressed
public void setCompressed(boolean isCompressed)
Description copied from interface:Buffer
Tags the buffer as compressed or uncompressed.- Specified by:
setCompressed
in interfaceBuffer
-
getDataType
public Buffer.DataType getDataType()
Description copied from interface:Buffer
Gets the type of data this buffer represents.- Specified by:
getDataType
in interfaceBuffer
-
setDataType
public void setDataType(Buffer.DataType dataType)
Description copied from interface:Buffer
Sets the type of data this buffer represents.- Specified by:
setDataType
in interfaceBuffer
-
deallocate
public void deallocate()
- Overrides:
deallocate
in classorg.apache.flink.shaded.netty4.io.netty.channel.DefaultFileRegion
-
readInto
public Buffer readInto(MemorySegment segment) throws IOException
- Throws:
IOException
-
-