Class IOManager
- java.lang.Object
-
- org.apache.flink.runtime.io.disk.iomanager.IOManager
-
- All Implemented Interfaces:
AutoCloseable
- Direct Known Subclasses:
IOManagerAsync
public abstract class IOManager extends Object implements AutoCloseable
The facade for the provided I/O manager services.
-
-
Field Summary
Fields Modifier and Type Field Description protected ExecutorService
executorService
protected static org.slf4j.Logger
LOG
-
Constructor Summary
Constructors Modifier Constructor Description protected
IOManager(String[] tempDirs, ExecutorService executorService)
Constructs a new IOManager.
-
Method Summary
All Methods Static Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
close()
Removes all temporary files.BlockChannelReader<MemorySegment>
createBlockChannelReader(FileIOChannel.ID channelID)
Creates a block channel reader that reads blocks from the given channel.abstract BlockChannelReader<MemorySegment>
createBlockChannelReader(FileIOChannel.ID channelID, LinkedBlockingQueue<MemorySegment> returnQueue)
Creates a block channel reader that reads blocks from the given channel.BlockChannelWriter<MemorySegment>
createBlockChannelWriter(FileIOChannel.ID channelID)
Creates a block channel writer that writes to the given channel.abstract BlockChannelWriter<MemorySegment>
createBlockChannelWriter(FileIOChannel.ID channelID, LinkedBlockingQueue<MemorySegment> returnQueue)
Creates a block channel writer that writes to the given channel.abstract BlockChannelWriterWithCallback<MemorySegment>
createBlockChannelWriter(FileIOChannel.ID channelID, RequestDoneCallback<MemorySegment> callback)
Creates a block channel writer that writes to the given channel.abstract BufferFileReader
createBufferFileReader(FileIOChannel.ID channelID, RequestDoneCallback<Buffer> callback)
abstract BufferFileSegmentReader
createBufferFileSegmentReader(FileIOChannel.ID channelID, RequestDoneCallback<FileSegment> callback)
abstract BufferFileWriter
createBufferFileWriter(FileIOChannel.ID channelID)
abstract BulkBlockChannelReader
createBulkBlockChannelReader(FileIOChannel.ID channelID, List<MemorySegment> targetSegments, int numBlocks)
Creates a block channel reader that reads all blocks from the given channel directly in one bulk.FileIOChannel.ID
createChannel()
Creates a newFileIOChannel.ID
in one of the temp directories.FileIOChannel.Enumerator
createChannelEnumerator()
Creates a newFileIOChannel.Enumerator
, spreading the channels in a round-robin fashion across the temporary file directories.static void
deleteChannel(FileIOChannel.ID channel)
Deletes the file underlying the given channel.ExecutorService
getExecutorService()
File[]
getSpillingDirectories()
Gets the directories that the I/O manager spills to.String[]
getSpillingDirectoriesPaths()
Gets the directories that the I/O manager spills to, as path strings.
-
-
-
Field Detail
-
LOG
protected static final org.slf4j.Logger LOG
-
executorService
protected final ExecutorService executorService
-
-
Constructor Detail
-
IOManager
protected IOManager(String[] tempDirs, ExecutorService executorService)
Constructs a new IOManager.- Parameters:
tempDirs
- The basic directories for files underlying anonymous channels.
-
-
Method Detail
-
close
public void close() throws Exception
Removes all temporary files.- Specified by:
close
in interfaceAutoCloseable
- Throws:
Exception
-
createChannel
public FileIOChannel.ID createChannel()
Creates a newFileIOChannel.ID
in one of the temp directories. Multiple invocations of this method spread the channels evenly across the different directories.- Returns:
- A channel to a temporary directory.
-
createChannelEnumerator
public FileIOChannel.Enumerator createChannelEnumerator()
Creates a newFileIOChannel.Enumerator
, spreading the channels in a round-robin fashion across the temporary file directories.- Returns:
- An enumerator for channels.
-
deleteChannel
public static void deleteChannel(FileIOChannel.ID channel)
Deletes the file underlying the given channel. If the channel is still open, this call may fail.- Parameters:
channel
- The channel to be deleted.
-
getSpillingDirectories
public File[] getSpillingDirectories()
Gets the directories that the I/O manager spills to.- Returns:
- The directories that the I/O manager spills to.
-
getSpillingDirectoriesPaths
public String[] getSpillingDirectoriesPaths()
Gets the directories that the I/O manager spills to, as path strings.- Returns:
- The directories that the I/O manager spills to, as path strings.
-
createBlockChannelWriter
public BlockChannelWriter<MemorySegment> createBlockChannelWriter(FileIOChannel.ID channelID) throws IOException
Creates a block channel writer that writes to the given channel. The writer adds the written segment to its return-queue afterwards (to allow for asynchronous implementations).- Parameters:
channelID
- The descriptor for the channel to write to.- Returns:
- A block channel writer that writes to the given channel.
- Throws:
IOException
- Thrown, if the channel for the writer could not be opened.
-
createBlockChannelWriter
public abstract BlockChannelWriter<MemorySegment> createBlockChannelWriter(FileIOChannel.ID channelID, LinkedBlockingQueue<MemorySegment> returnQueue) throws IOException
Creates a block channel writer that writes to the given channel. The writer adds the written segment to the given queue (to allow for asynchronous implementations).- Parameters:
channelID
- The descriptor for the channel to write to.returnQueue
- The queue to put the written buffers into.- Returns:
- A block channel writer that writes to the given channel.
- Throws:
IOException
- Thrown, if the channel for the writer could not be opened.
-
createBlockChannelWriter
public abstract BlockChannelWriterWithCallback<MemorySegment> createBlockChannelWriter(FileIOChannel.ID channelID, RequestDoneCallback<MemorySegment> callback) throws IOException
Creates a block channel writer that writes to the given channel. The writer calls the given callback after the I/O operation has been performed (successfully or unsuccessfully), to allow for asynchronous implementations.- Parameters:
channelID
- The descriptor for the channel to write to.callback
- The callback to be called for- Returns:
- A block channel writer that writes to the given channel.
- Throws:
IOException
- Thrown, if the channel for the writer could not be opened.
-
createBlockChannelReader
public BlockChannelReader<MemorySegment> createBlockChannelReader(FileIOChannel.ID channelID) throws IOException
Creates a block channel reader that reads blocks from the given channel. The reader pushed full memory segments (with the read data) to its "return queue", to allow for asynchronous read implementations.- Parameters:
channelID
- The descriptor for the channel to write to.- Returns:
- A block channel reader that reads from the given channel.
- Throws:
IOException
- Thrown, if the channel for the reader could not be opened.
-
createBlockChannelReader
public abstract BlockChannelReader<MemorySegment> createBlockChannelReader(FileIOChannel.ID channelID, LinkedBlockingQueue<MemorySegment> returnQueue) throws IOException
Creates a block channel reader that reads blocks from the given channel. The reader pushes the full segments to the given queue, to allow for asynchronous implementations.- Parameters:
channelID
- The descriptor for the channel to write to.returnQueue
- The queue to put the full buffers into.- Returns:
- A block channel reader that reads from the given channel.
- Throws:
IOException
- Thrown, if the channel for the reader could not be opened.
-
createBufferFileWriter
public abstract BufferFileWriter createBufferFileWriter(FileIOChannel.ID channelID) throws IOException
- Throws:
IOException
-
createBufferFileReader
public abstract BufferFileReader createBufferFileReader(FileIOChannel.ID channelID, RequestDoneCallback<Buffer> callback) throws IOException
- Throws:
IOException
-
createBufferFileSegmentReader
public abstract BufferFileSegmentReader createBufferFileSegmentReader(FileIOChannel.ID channelID, RequestDoneCallback<FileSegment> callback) throws IOException
- Throws:
IOException
-
createBulkBlockChannelReader
public abstract BulkBlockChannelReader createBulkBlockChannelReader(FileIOChannel.ID channelID, List<MemorySegment> targetSegments, int numBlocks) throws IOException
Creates a block channel reader that reads all blocks from the given channel directly in one bulk. The reader draws segments to read the blocks into from a supplied list, which must contain as many segments as the channel has blocks. After the reader is done, the list with the full segments can be obtained from the reader.If a channel is not to be read in one bulk, but in multiple smaller batches, a
BlockChannelReader
should be used.- Parameters:
channelID
- The descriptor for the channel to write to.targetSegments
- The list to take the segments from into which to read the data.numBlocks
- The number of blocks in the channel to read.- Returns:
- A block channel reader that reads from the given channel.
- Throws:
IOException
- Thrown, if the channel for the reader could not be opened.
-
getExecutorService
public ExecutorService getExecutorService()
-
-