Class AsynchronousBlockReader

  • All Implemented Interfaces:
    BlockChannelReader<MemorySegment>, FileIOChannel

    public class AsynchronousBlockReader
    extends AsynchronousFileIOChannel<MemorySegment,​org.apache.flink.runtime.io.disk.iomanager.ReadRequest>
    implements BlockChannelReader<MemorySegment>
    A reader that reads data in blocks from a file channel. The reader reads the blocks into a MemorySegment in an asynchronous fashion. That is, a read request is not processed by the thread that issues it, but by an asynchronous reader thread. Once the read request is done, the asynchronous reader adds the full MemorySegment to a return queue where it can be popped by the worker thread, once it needs the data. The return queue is in this case a LinkedBlockingQueue, such that the working thread blocks until the request has been served, if the request is still pending when the it requires the data.

    Typical pre-fetching reads are done by issuing the read requests early and popping the return queue once the data is actually needed.

    The reader has no notion whether the size of the memory segments is actually the size of the blocks on disk, or even whether the file was written in blocks of the same size, or in blocks at all. Ensuring that the writing and reading is consistent with each other (same blocks sizes) is up to the programmer.

    • Constructor Detail

      • AsynchronousBlockReader

        protected AsynchronousBlockReader​(FileIOChannel.ID channelID,
                                          RequestQueue<org.apache.flink.runtime.io.disk.iomanager.ReadRequest> requestQueue,
                                          LinkedBlockingQueue<MemorySegment> returnSegments)
                                   throws IOException
        Creates a new block channel reader for the given channel.
        Parameters:
        channelID - The ID of the channel to read.
        requestQueue - The request queue of the asynchronous reader thread, to which the I/O requests are added.
        returnSegments - The return queue, to which the full Memory Segments are added.
        Throws:
        IOException - Thrown, if the underlying file channel could not be opened.
    • Method Detail

      • readBlock

        public void readBlock​(MemorySegment segment)
                       throws IOException
        Issues a read request, which will asynchronously fill the given segment with the next block in the underlying file channel. Once the read request is fulfilled, the segment will be added to this reader's return queue.
        Specified by:
        readBlock in interface BlockChannelReader<MemorySegment>
        Parameters:
        segment - The segment to read the block into.
        Throws:
        IOException - Thrown, when the reader encounters an I/O error. Due to the asynchronous nature of the reader, the exception thrown here may have been caused by an earlier read request.
      • getNextReturnedBlock

        public MemorySegment getNextReturnedBlock()
                                           throws IOException
        Gets the next memory segment that has been filled with data by the reader. This method blocks until such a segment is available, or until an error occurs in the reader, or the reader is closed.

        WARNING: If this method is invoked without any segment ever returning (for example, because the readBlock(MemorySegment) method has not been invoked appropriately), the method may block forever.

        Specified by:
        getNextReturnedBlock in interface BlockChannelReader<MemorySegment>
        Returns:
        The next memory segment from the reader's return queue.
        Throws:
        IOException - Thrown, if an I/O error occurs in the reader while waiting for the request to return.