|
@@ -18,9 +18,11 @@
|
|
|
|
|
|
package org.apache.hadoop.fs.azurebfs.services;
|
|
package org.apache.hadoop.fs.azurebfs.services;
|
|
|
|
|
|
|
|
+import java.io.FileNotFoundException;
|
|
import java.io.IOException;
|
|
import java.io.IOException;
|
|
import java.io.InterruptedIOException;
|
|
import java.io.InterruptedIOException;
|
|
import java.io.OutputStream;
|
|
import java.io.OutputStream;
|
|
|
|
+import java.net.HttpURLConnection;
|
|
import java.util.Locale;
|
|
import java.util.Locale;
|
|
import java.util.concurrent.ConcurrentLinkedDeque;
|
|
import java.util.concurrent.ConcurrentLinkedDeque;
|
|
import java.util.concurrent.LinkedBlockingQueue;
|
|
import java.util.concurrent.LinkedBlockingQueue;
|
|
@@ -33,10 +35,11 @@ import java.util.concurrent.TimeUnit;
|
|
import com.google.common.annotations.VisibleForTesting;
|
|
import com.google.common.annotations.VisibleForTesting;
|
|
import com.google.common.base.Preconditions;
|
|
import com.google.common.base.Preconditions;
|
|
|
|
|
|
|
|
+import org.apache.hadoop.fs.azurebfs.contracts.exceptions.AbfsRestOperationException;
|
|
|
|
+import org.apache.hadoop.fs.azurebfs.contracts.exceptions.AzureBlobFileSystemException;
|
|
import org.apache.hadoop.fs.FSExceptionMessages;
|
|
import org.apache.hadoop.fs.FSExceptionMessages;
|
|
import org.apache.hadoop.fs.StreamCapabilities;
|
|
import org.apache.hadoop.fs.StreamCapabilities;
|
|
import org.apache.hadoop.fs.Syncable;
|
|
import org.apache.hadoop.fs.Syncable;
|
|
-import org.apache.hadoop.fs.azurebfs.contracts.exceptions.AzureBlobFileSystemException;
|
|
|
|
|
|
|
|
/**
|
|
/**
|
|
* The BlobFsOutputStream for Rest AbfsClient.
|
|
* The BlobFsOutputStream for Rest AbfsClient.
|
|
@@ -290,6 +293,12 @@ public class AbfsOutputStream extends OutputStream implements Syncable, StreamCa
|
|
try {
|
|
try {
|
|
writeOperation.task.get();
|
|
writeOperation.task.get();
|
|
} catch (Exception ex) {
|
|
} catch (Exception ex) {
|
|
|
|
+ if (ex.getCause() instanceof AbfsRestOperationException) {
|
|
|
|
+ if (((AbfsRestOperationException) ex.getCause()).getStatusCode() == HttpURLConnection.HTTP_NOT_FOUND) {
|
|
|
|
+ throw new FileNotFoundException(ex.getMessage());
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
if (ex.getCause() instanceof AzureBlobFileSystemException) {
|
|
if (ex.getCause() instanceof AzureBlobFileSystemException) {
|
|
ex = (AzureBlobFileSystemException) ex.getCause();
|
|
ex = (AzureBlobFileSystemException) ex.getCause();
|
|
}
|
|
}
|
|
@@ -313,6 +322,11 @@ public class AbfsOutputStream extends OutputStream implements Syncable, StreamCa
|
|
try {
|
|
try {
|
|
client.flush(path, offset, retainUncommitedData);
|
|
client.flush(path, offset, retainUncommitedData);
|
|
} catch (AzureBlobFileSystemException ex) {
|
|
} catch (AzureBlobFileSystemException ex) {
|
|
|
|
+ if (ex instanceof AbfsRestOperationException) {
|
|
|
|
+ if (((AbfsRestOperationException) ex).getStatusCode() == HttpURLConnection.HTTP_NOT_FOUND) {
|
|
|
|
+ throw new FileNotFoundException(ex.getMessage());
|
|
|
|
+ }
|
|
|
|
+ }
|
|
throw new IOException(ex);
|
|
throw new IOException(ex);
|
|
}
|
|
}
|
|
this.lastFlushOffset = offset;
|
|
this.lastFlushOffset = offset;
|