|
@@ -40,6 +40,7 @@ import org.apache.hadoop.hdfs.server.namenode.NotReplicatedYetException;
|
|
import org.apache.hadoop.hdfs.server.namenode.SafeModeException;
|
|
import org.apache.hadoop.hdfs.server.namenode.SafeModeException;
|
|
import org.apache.hadoop.io.EnumSetWritable;
|
|
import org.apache.hadoop.io.EnumSetWritable;
|
|
import org.apache.hadoop.io.Text;
|
|
import org.apache.hadoop.io.Text;
|
|
|
|
+import org.apache.hadoop.io.retry.AtMostOnce;
|
|
import org.apache.hadoop.io.retry.Idempotent;
|
|
import org.apache.hadoop.io.retry.Idempotent;
|
|
import org.apache.hadoop.security.AccessControlException;
|
|
import org.apache.hadoop.security.AccessControlException;
|
|
import org.apache.hadoop.security.KerberosInfo;
|
|
import org.apache.hadoop.security.KerberosInfo;
|
|
@@ -139,7 +140,7 @@ public interface ClientProtocol {
|
|
* <p>
|
|
* <p>
|
|
* Blocks have a maximum size. Clients that intend to create
|
|
* Blocks have a maximum size. Clients that intend to create
|
|
* multi-block files must also use
|
|
* multi-block files must also use
|
|
- * {@link #addBlock(String, String, ExtendedBlock, DatanodeInfo[])}
|
|
|
|
|
|
+ * {@link #addBlock}
|
|
*
|
|
*
|
|
* @param src path of the file being created.
|
|
* @param src path of the file being created.
|
|
* @param masked masked permission.
|
|
* @param masked masked permission.
|
|
@@ -170,7 +171,10 @@ public interface ClientProtocol {
|
|
*
|
|
*
|
|
* RuntimeExceptions:
|
|
* RuntimeExceptions:
|
|
* @throws InvalidPathException Path <code>src</code> is invalid
|
|
* @throws InvalidPathException Path <code>src</code> is invalid
|
|
|
|
+ * <p>
|
|
|
|
+ * <em>Note that create with {@link CreateFlag#OVERWRITE} is idempotent.</em>
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public HdfsFileStatus create(String src, FsPermission masked,
|
|
public HdfsFileStatus create(String src, FsPermission masked,
|
|
String clientName, EnumSetWritable<CreateFlag> flag,
|
|
String clientName, EnumSetWritable<CreateFlag> flag,
|
|
boolean createParent, short replication, long blockSize)
|
|
boolean createParent, short replication, long blockSize)
|
|
@@ -204,6 +208,7 @@ public interface ClientProtocol {
|
|
* RuntimeExceptions:
|
|
* RuntimeExceptions:
|
|
* @throws UnsupportedOperationException if append is not supported
|
|
* @throws UnsupportedOperationException if append is not supported
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public LocatedBlock append(String src, String clientName)
|
|
public LocatedBlock append(String src, String clientName)
|
|
throws AccessControlException, DSQuotaExceededException,
|
|
throws AccessControlException, DSQuotaExceededException,
|
|
FileNotFoundException, SafeModeException, UnresolvedLinkException,
|
|
FileNotFoundException, SafeModeException, UnresolvedLinkException,
|
|
@@ -409,6 +414,7 @@ public interface ClientProtocol {
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws IOException an I/O error occurred
|
|
* @throws IOException an I/O error occurred
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public boolean rename(String src, String dst)
|
|
public boolean rename(String src, String dst)
|
|
throws UnresolvedLinkException, SnapshotAccessControlException, IOException;
|
|
throws UnresolvedLinkException, SnapshotAccessControlException, IOException;
|
|
|
|
|
|
@@ -422,6 +428,7 @@ public interface ClientProtocol {
|
|
* contains a symlink
|
|
* contains a symlink
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public void concat(String trg, String[] srcs)
|
|
public void concat(String trg, String[] srcs)
|
|
throws IOException, UnresolvedLinkException, SnapshotAccessControlException;
|
|
throws IOException, UnresolvedLinkException, SnapshotAccessControlException;
|
|
|
|
|
|
@@ -460,6 +467,7 @@ public interface ClientProtocol {
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws IOException If an I/O error occurred
|
|
* @throws IOException If an I/O error occurred
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public void rename2(String src, String dst, Options.Rename... options)
|
|
public void rename2(String src, String dst, Options.Rename... options)
|
|
throws AccessControlException, DSQuotaExceededException,
|
|
throws AccessControlException, DSQuotaExceededException,
|
|
FileAlreadyExistsException, FileNotFoundException,
|
|
FileAlreadyExistsException, FileNotFoundException,
|
|
@@ -484,6 +492,7 @@ public interface ClientProtocol {
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws IOException If an I/O error occurred
|
|
* @throws IOException If an I/O error occurred
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public boolean delete(String src, boolean recursive)
|
|
public boolean delete(String src, boolean recursive)
|
|
throws AccessControlException, FileNotFoundException, SafeModeException,
|
|
throws AccessControlException, FileNotFoundException, SafeModeException,
|
|
UnresolvedLinkException, SnapshotAccessControlException, IOException;
|
|
UnresolvedLinkException, SnapshotAccessControlException, IOException;
|
|
@@ -704,6 +713,7 @@ public interface ClientProtocol {
|
|
* @throws AccessControlException if the superuser privilege is violated.
|
|
* @throws AccessControlException if the superuser privilege is violated.
|
|
* @throws IOException if image creation failed.
|
|
* @throws IOException if image creation failed.
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public void saveNamespace() throws AccessControlException, IOException;
|
|
public void saveNamespace() throws AccessControlException, IOException;
|
|
|
|
|
|
|
|
|
|
@@ -725,6 +735,7 @@ public interface ClientProtocol {
|
|
*
|
|
*
|
|
* @throws AccessControlException if the superuser privilege is violated.
|
|
* @throws AccessControlException if the superuser privilege is violated.
|
|
*/
|
|
*/
|
|
|
|
+ @Idempotent
|
|
public boolean restoreFailedStorage(String arg)
|
|
public boolean restoreFailedStorage(String arg)
|
|
throws AccessControlException, IOException;
|
|
throws AccessControlException, IOException;
|
|
|
|
|
|
@@ -732,6 +743,7 @@ public interface ClientProtocol {
|
|
* Tells the namenode to reread the hosts and exclude files.
|
|
* Tells the namenode to reread the hosts and exclude files.
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @Idempotent
|
|
public void refreshNodes() throws IOException;
|
|
public void refreshNodes() throws IOException;
|
|
|
|
|
|
/**
|
|
/**
|
|
@@ -741,6 +753,7 @@ public interface ClientProtocol {
|
|
*
|
|
*
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @Idempotent
|
|
public void finalizeUpgrade() throws IOException;
|
|
public void finalizeUpgrade() throws IOException;
|
|
|
|
|
|
/**
|
|
/**
|
|
@@ -763,6 +776,7 @@ public interface ClientProtocol {
|
|
*
|
|
*
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @Idempotent
|
|
public void metaSave(String filename) throws IOException;
|
|
public void metaSave(String filename) throws IOException;
|
|
|
|
|
|
/**
|
|
/**
|
|
@@ -918,6 +932,7 @@ public interface ClientProtocol {
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws SnapshotAccessControlException if path is in RO snapshot
|
|
* @throws IOException If an I/O error occurred
|
|
* @throws IOException If an I/O error occurred
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public void createSymlink(String target, String link, FsPermission dirPerm,
|
|
public void createSymlink(String target, String link, FsPermission dirPerm,
|
|
boolean createParent) throws AccessControlException,
|
|
boolean createParent) throws AccessControlException,
|
|
FileAlreadyExistsException, FileNotFoundException,
|
|
FileAlreadyExistsException, FileNotFoundException,
|
|
@@ -965,6 +980,7 @@ public interface ClientProtocol {
|
|
* @param newNodes datanodes in the pipeline
|
|
* @param newNodes datanodes in the pipeline
|
|
* @throws IOException if any error occurs
|
|
* @throws IOException if any error occurs
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public void updatePipeline(String clientName, ExtendedBlock oldBlock,
|
|
public void updatePipeline(String clientName, ExtendedBlock oldBlock,
|
|
ExtendedBlock newBlock, DatanodeID[] newNodes)
|
|
ExtendedBlock newBlock, DatanodeID[] newNodes)
|
|
throws IOException;
|
|
throws IOException;
|
|
@@ -997,6 +1013,7 @@ public interface ClientProtocol {
|
|
* @param token delegation token
|
|
* @param token delegation token
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @Idempotent
|
|
public void cancelDelegationToken(Token<DelegationTokenIdentifier> token)
|
|
public void cancelDelegationToken(Token<DelegationTokenIdentifier> token)
|
|
throws IOException;
|
|
throws IOException;
|
|
|
|
|
|
@@ -1005,6 +1022,7 @@ public interface ClientProtocol {
|
|
* DataTransferProtocol to/from DataNodes.
|
|
* DataTransferProtocol to/from DataNodes.
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @Idempotent
|
|
public DataEncryptionKey getDataEncryptionKey() throws IOException;
|
|
public DataEncryptionKey getDataEncryptionKey() throws IOException;
|
|
|
|
|
|
/**
|
|
/**
|
|
@@ -1014,6 +1032,7 @@ public interface ClientProtocol {
|
|
* @return the snapshot path.
|
|
* @return the snapshot path.
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public String createSnapshot(String snapshotRoot, String snapshotName)
|
|
public String createSnapshot(String snapshotRoot, String snapshotName)
|
|
throws IOException;
|
|
throws IOException;
|
|
|
|
|
|
@@ -1023,6 +1042,7 @@ public interface ClientProtocol {
|
|
* @param snapshotName Name of the snapshot for the snapshottable directory
|
|
* @param snapshotName Name of the snapshot for the snapshottable directory
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public void deleteSnapshot(String snapshotRoot, String snapshotName)
|
|
public void deleteSnapshot(String snapshotRoot, String snapshotName)
|
|
throws IOException;
|
|
throws IOException;
|
|
|
|
|
|
@@ -1033,6 +1053,7 @@ public interface ClientProtocol {
|
|
* @param snapshotNewName new name of the snapshot
|
|
* @param snapshotNewName new name of the snapshot
|
|
* @throws IOException
|
|
* @throws IOException
|
|
*/
|
|
*/
|
|
|
|
+ @AtMostOnce
|
|
public void renameSnapshot(String snapshotRoot, String snapshotOldName,
|
|
public void renameSnapshot(String snapshotRoot, String snapshotOldName,
|
|
String snapshotNewName) throws IOException;
|
|
String snapshotNewName) throws IOException;
|
|
|
|
|