Class HashPartition<BT,PT>
- java.lang.Object
-
- org.apache.flink.runtime.memory.AbstractPagedInputView
-
- org.apache.flink.runtime.operators.hash.HashPartition<BT,PT>
-
- Type Parameters:
BT
- The type of the build side records.PT
- The type of the probe side records.
- All Implemented Interfaces:
DataInput
,DataInputView
,SeekableDataInputView
- Direct Known Subclasses:
ReOpenableHashPartition
public class HashPartition<BT,PT> extends AbstractPagedInputView implements SeekableDataInputView
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description protected static class
HashPartition.BuildSideBuffer
-
Field Summary
Fields Modifier and Type Field Description protected boolean
furtherPartitioning
protected int
nextOverflowBucket
protected int
numOverflowSegments
protected MemorySegment[]
overflowSegments
protected MemorySegment[]
partitionBuffers
protected ChannelWriterOutputView
probeSideBuffer
protected BlockChannelWriter<MemorySegment>
probeSideChannel
protected long
probeSideRecordCounter
protected int
recursionLevel
-
Fields inherited from class org.apache.flink.runtime.memory.AbstractPagedInputView
headerLength
-
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description void
clearAllMemory(List<MemorySegment> target)
void
finalizeBuildPhase(IOManager ioAccess, FileIOChannel.Enumerator probeChannelEnumerator, LinkedBlockingQueue<MemorySegment> bufferReturnQueue)
int
finalizeProbePhase(List<MemorySegment> freeMemory, List<HashPartition<BT,PT>> spilledPartitions, boolean keepUnprobedSpilledPartitions)
int
getBuildSideBlockCount()
BlockChannelWriter<MemorySegment>
getBuildSideChannel()
long
getBuildSideRecordCount()
protected int
getLimitForSegment(MemorySegment segment)
Gets the limit for reading bytes from the given memory segment.int
getNumOccupiedMemorySegments()
Gets the number of memory segments used by this partition, which includes build side memory buffers and overflow memory segments.int
getPartitionNumber()
Gets the partition number of this partition.int
getProbeSideBlockCount()
BlockChannelWriter<MemorySegment>
getProbeSideChannel()
long
getProbeSideRecordCount()
int
getRecursionLevel()
Gets this partition's recursion level.long
insertIntoBuildBuffer(BT record)
Inserts the given object into the current buffer.void
insertIntoProbeBuffer(PT record)
Inserts the given record into the probe side buffers.boolean
isInMemory()
Checks whether this partition is in memory or spilled.protected MemorySegment
nextSegment(MemorySegment current)
The method by which concrete subclasses realize page crossing.void
prepareProbePhase(IOManager ioAccess, FileIOChannel.Enumerator probeChannelEnumerator, LinkedBlockingQueue<MemorySegment> bufferReturnQueue)
protected void
setFurtherPatitioning(boolean v)
void
setReadPosition(long pointer)
Sets the read pointer to the given position.int
spillPartition(List<MemorySegment> target, IOManager ioAccess, FileIOChannel.ID targetChannel, LinkedBlockingQueue<MemorySegment> bufferReturnQueue)
Spills this partition to disk and sets it up such that it continues spilling records that are added to it.-
Methods inherited from class org.apache.flink.runtime.memory.AbstractPagedInputView
advance, clear, doAdvance, getCurrentPositionInSegment, getCurrentSegment, getCurrentSegmentLimit, getHeaderLength, read, read, readBoolean, readByte, readChar, readDouble, readFloat, readFully, readFully, readInt, readLine, readLong, readShort, readUnsignedByte, readUnsignedShort, readUTF, seekInput, skipBytes, skipBytesToRead
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface java.io.DataInput
readBoolean, readByte, readChar, readDouble, readFloat, readFully, readFully, readInt, readLine, readLong, readShort, readUnsignedByte, readUnsignedShort, readUTF, skipBytes
-
Methods inherited from interface org.apache.flink.core.memory.DataInputView
read, read, skipBytesToRead
-
-
-
-
Field Detail
-
overflowSegments
protected MemorySegment[] overflowSegments
-
numOverflowSegments
protected int numOverflowSegments
-
nextOverflowBucket
protected int nextOverflowBucket
-
partitionBuffers
protected MemorySegment[] partitionBuffers
-
probeSideBuffer
protected ChannelWriterOutputView probeSideBuffer
-
probeSideRecordCounter
protected long probeSideRecordCounter
-
recursionLevel
protected int recursionLevel
-
probeSideChannel
protected BlockChannelWriter<MemorySegment> probeSideChannel
-
furtherPartitioning
protected boolean furtherPartitioning
-
-
Method Detail
-
setFurtherPatitioning
protected void setFurtherPatitioning(boolean v)
-
getPartitionNumber
public int getPartitionNumber()
Gets the partition number of this partition.- Returns:
- This partition's number.
-
getRecursionLevel
public int getRecursionLevel()
Gets this partition's recursion level.- Returns:
- The partition's recursion level.
-
isInMemory
public final boolean isInMemory()
Checks whether this partition is in memory or spilled.- Returns:
- True, if the partition is in memory, false if it is spilled.
-
getNumOccupiedMemorySegments
public int getNumOccupiedMemorySegments()
Gets the number of memory segments used by this partition, which includes build side memory buffers and overflow memory segments.- Returns:
- The number of occupied memory segments.
-
getBuildSideBlockCount
public int getBuildSideBlockCount()
-
getProbeSideBlockCount
public int getProbeSideBlockCount()
-
getBuildSideRecordCount
public long getBuildSideRecordCount()
-
getProbeSideRecordCount
public long getProbeSideRecordCount()
-
getBuildSideChannel
public BlockChannelWriter<MemorySegment> getBuildSideChannel()
-
getProbeSideChannel
public BlockChannelWriter<MemorySegment> getProbeSideChannel()
-
insertIntoBuildBuffer
public final long insertIntoBuildBuffer(BT record) throws IOException
Inserts the given object into the current buffer. This method returns a pointer that can be used to address the written record in this partition, if it is in-memory. The returned pointers have no expressiveness in the case where the partition is spilled.- Parameters:
record
- The object to be written to the partition.- Returns:
- A pointer to the object in the partition, or
-1
, if the partition is spilled. - Throws:
IOException
- Thrown, when this is a spilled partition and the write failed.
-
insertIntoProbeBuffer
public final void insertIntoProbeBuffer(PT record) throws IOException
Inserts the given record into the probe side buffers. This method is only applicable when the partition was spilled while processing the build side.If this method is invoked when the partition is still being built, it has undefined behavior.
- Parameters:
record
- The record to be inserted into the probe side buffers.- Throws:
IOException
- Thrown, if the buffer is full, needs to be spilled, and spilling causes an error.
-
spillPartition
public int spillPartition(List<MemorySegment> target, IOManager ioAccess, FileIOChannel.ID targetChannel, LinkedBlockingQueue<MemorySegment> bufferReturnQueue) throws IOException
Spills this partition to disk and sets it up such that it continues spilling records that are added to it. The spilling process must free at least one buffer, either in the partition's record buffers, or in the memory segments for overflow buckets. The partition immediately takes back one buffer to use it for further spilling.- Parameters:
target
- The list to which memory segments from overflow buckets are added.ioAccess
- The I/O manager to be used to create a writer to disk.targetChannel
- The id of the target channel for this partition.- Returns:
- The number of buffers that were freed by spilling this partition.
- Throws:
IOException
- Thrown, if the writing failed.
-
finalizeBuildPhase
public void finalizeBuildPhase(IOManager ioAccess, FileIOChannel.Enumerator probeChannelEnumerator, LinkedBlockingQueue<MemorySegment> bufferReturnQueue) throws IOException
- Throws:
IOException
-
finalizeProbePhase
public int finalizeProbePhase(List<MemorySegment> freeMemory, List<HashPartition<BT,PT>> spilledPartitions, boolean keepUnprobedSpilledPartitions) throws IOException
- Parameters:
keepUnprobedSpilledPartitions
- If true then partitions that were spilled but received no further probe requests will be retained; used for build-side outer joins.- Returns:
- The number of write-behind buffers reclaimable after this method call.
- Throws:
IOException
-
clearAllMemory
public void clearAllMemory(List<MemorySegment> target)
-
prepareProbePhase
public void prepareProbePhase(IOManager ioAccess, FileIOChannel.Enumerator probeChannelEnumerator, LinkedBlockingQueue<MemorySegment> bufferReturnQueue) throws IOException
- Throws:
IOException
-
setReadPosition
public void setReadPosition(long pointer)
Description copied from interface:SeekableDataInputView
Sets the read pointer to the given position.- Specified by:
setReadPosition
in interfaceSeekableDataInputView
- Parameters:
pointer
- The new read position.
-
nextSegment
protected MemorySegment nextSegment(MemorySegment current) throws IOException
Description copied from class:AbstractPagedInputView
The method by which concrete subclasses realize page crossing. This method is invoked when the current page is exhausted and a new page is required to continue the reading. If no further page is available, this method must throw anEOFException
.- Specified by:
nextSegment
in classAbstractPagedInputView
- Parameters:
current
- The current page that was read to its limit. May benull
, if this method is invoked for the first time.- Returns:
- The next page from which the reading should continue. May not be
null
. If the input is exhausted, anEOFException
must be thrown instead. - Throws:
EOFException
- Thrown, if no further segment is available.IOException
- Thrown, if the method cannot provide the next page due to an I/O related problem.
-
getLimitForSegment
protected int getLimitForSegment(MemorySegment segment)
Description copied from class:AbstractPagedInputView
Gets the limit for reading bytes from the given memory segment. This method must return the position of the byte after the last valid byte in the given memory segment. When the position returned by this method is reached, the view will attempt to switch to the next memory segment.- Specified by:
getLimitForSegment
in classAbstractPagedInputView
- Parameters:
segment
- The segment to determine the limit for.- Returns:
- The limit for the given memory segment.
-
-