Package org.apache.flink.core.fs
Class RecoverableFsDataOutputStream
- java.lang.Object
-
- java.io.OutputStream
-
- org.apache.flink.core.fs.FSDataOutputStream
-
- org.apache.flink.core.fs.RecoverableFsDataOutputStream
-
- All Implemented Interfaces:
Closeable,Flushable,AutoCloseable
- Direct Known Subclasses:
CommitterFromPersistRecoverableFsDataOutputStream
@PublicEvolving public abstract class RecoverableFsDataOutputStream extends FSDataOutputStream
An output stream to a file system that can be recovered at well defined points. The stream initially writes to hidden files or temp files and only creates the target file once it is closed and "committed".
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static interfaceRecoverableFsDataOutputStream.CommitterA committer can publish the file of a stream that was closed.
-
Constructor Summary
Constructors Constructor Description RecoverableFsDataOutputStream()
-
Method Summary
All Methods Instance Methods Abstract Methods Modifier and Type Method Description abstract voidclose()Closes this stream.abstract RecoverableFsDataOutputStream.CommittercloseForCommit()Closes the stream, ensuring persistence of all data (similar toFSDataOutputStream.sync()).abstract RecoverableWriter.ResumeRecoverablepersist()Ensures all data so far is persistent (similar toFSDataOutputStream.sync()) and returns a handle to recover the stream at the current position.-
Methods inherited from class org.apache.flink.core.fs.FSDataOutputStream
flush, getPos, sync
-
Methods inherited from class java.io.OutputStream
nullOutputStream, write, write, write
-
-
-
-
Method Detail
-
persist
public abstract RecoverableWriter.ResumeRecoverable persist() throws IOException
Ensures all data so far is persistent (similar toFSDataOutputStream.sync()) and returns a handle to recover the stream at the current position.- Throws:
IOException
-
closeForCommit
public abstract RecoverableFsDataOutputStream.Committer closeForCommit() throws IOException
Closes the stream, ensuring persistence of all data (similar toFSDataOutputStream.sync()). This returns a Committer that can be used to publish (make visible) the file that the stream was writing to.- Throws:
IOException
-
close
public abstract void close() throws IOExceptionCloses this stream. Closing the steam releases the local resources that the stream uses, but does NOT result in durability of previously written data. This method should be interpreted as a "close in order to dispose" or "close on failure".In order to persist all previously written data, one needs to call the
closeForCommit()method and callRecoverableFsDataOutputStream.Committer.commit()on the returned committer object.- Specified by:
closein interfaceAutoCloseable- Specified by:
closein interfaceCloseable- Specified by:
closein classFSDataOutputStream- Throws:
IOException- Thrown if an error occurred during closing.
-
-