|
@@ -20,16 +20,30 @@ package org.apache.hadoop.fs.s3a;
|
|
|
|
|
|
import java.io.FileNotFoundException;
|
|
|
import java.io.IOException;
|
|
|
+import java.io.InputStream;
|
|
|
+import java.util.stream.Stream;
|
|
|
+
|
|
|
+import com.amazonaws.services.s3.AmazonS3;
|
|
|
+import com.amazonaws.services.s3.model.ListObjectsV2Request;
|
|
|
+import com.amazonaws.services.s3.model.ListObjectsV2Result;
|
|
|
+import com.amazonaws.services.s3.model.PutObjectRequest;
|
|
|
+import com.amazonaws.services.s3.model.S3ObjectSummary;
|
|
|
+import org.assertj.core.api.Assertions;
|
|
|
+import org.junit.Test;
|
|
|
|
|
|
import org.apache.hadoop.fs.Path;
|
|
|
+import org.apache.hadoop.fs.s3a.impl.StoreContext;
|
|
|
+import org.apache.hadoop.fs.s3a.s3guard.DDBPathMetadata;
|
|
|
+import org.apache.hadoop.fs.s3a.s3guard.DynamoDBMetadataStore;
|
|
|
import org.apache.hadoop.fs.s3a.s3guard.MetadataStore;
|
|
|
import org.apache.hadoop.fs.s3a.s3guard.NullMetadataStore;
|
|
|
-import org.junit.Assume;
|
|
|
-import org.junit.Test;
|
|
|
|
|
|
import static org.apache.hadoop.fs.contract.ContractTestUtils.assertRenameOutcome;
|
|
|
import static org.apache.hadoop.fs.contract.ContractTestUtils.touch;
|
|
|
import static org.apache.hadoop.test.LambdaTestUtils.intercept;
|
|
|
+import static org.apache.hadoop.fs.s3a.S3ATestUtils.assume;
|
|
|
+import static org.apache.hadoop.fs.s3a.S3ATestUtils.assumeFilesystemHasMetadatastore;
|
|
|
+import static org.apache.hadoop.fs.s3a.S3ATestUtils.getStatusWithEmptyDirFlag;
|
|
|
|
|
|
/**
|
|
|
* Test logic around whether or not a directory is empty, with S3Guard enabled.
|
|
@@ -84,7 +98,7 @@ public class ITestS3GuardEmptyDirs extends AbstractS3ATestBase {
|
|
|
@Test
|
|
|
public void testEmptyDirs() throws Exception {
|
|
|
S3AFileSystem fs = getFileSystem();
|
|
|
- Assume.assumeTrue(fs.hasMetadataStore());
|
|
|
+ assumeFilesystemHasMetadatastore(getFileSystem());
|
|
|
MetadataStore configuredMs = fs.getMetadataStore();
|
|
|
Path existingDir = path("existing-dir");
|
|
|
Path existingFile = path("existing-dir/existing-file");
|
|
@@ -126,4 +140,139 @@ public class ITestS3GuardEmptyDirs extends AbstractS3ATestBase {
|
|
|
configuredMs.forgetMetadata(existingDir);
|
|
|
}
|
|
|
}
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Test tombstones don't get in the way of a listing of the
|
|
|
+ * root dir.
|
|
|
+ * This test needs to create a path which appears first in the listing,
|
|
|
+ * and an entry which can come later. To allow the test to proceed
|
|
|
+ * while other tests are running, the filename "0000" is used for that
|
|
|
+ * deleted entry.
|
|
|
+ */
|
|
|
+ @Test
|
|
|
+ public void testTombstonesAndEmptyDirectories() throws Throwable {
|
|
|
+ S3AFileSystem fs = getFileSystem();
|
|
|
+ assumeFilesystemHasMetadatastore(getFileSystem());
|
|
|
+
|
|
|
+ // Create the first and last files.
|
|
|
+ Path base = path(getMethodName());
|
|
|
+ // use something ahead of all the ASCII alphabet characters so
|
|
|
+ // even during parallel test runs, this test is expected to work.
|
|
|
+ String first = "0000";
|
|
|
+ Path firstPath = new Path(base, first);
|
|
|
+
|
|
|
+ // this path is near the bottom of the ASCII string space.
|
|
|
+ // This isn't so critical.
|
|
|
+ String last = "zzzz";
|
|
|
+ Path lastPath = new Path(base, last);
|
|
|
+ touch(fs, firstPath);
|
|
|
+ touch(fs, lastPath);
|
|
|
+ // Delete first entry (+assert tombstone)
|
|
|
+ assertDeleted(firstPath, false);
|
|
|
+ DynamoDBMetadataStore ddbMs = getRequiredDDBMetastore(fs);
|
|
|
+ DDBPathMetadata firstMD = ddbMs.get(firstPath);
|
|
|
+ assertNotNull("No MD for " + firstPath, firstMD);
|
|
|
+ assertTrue("Not a tombstone " + firstMD,
|
|
|
+ firstMD.isDeleted());
|
|
|
+ // PUT child to store going past the FS entirely.
|
|
|
+ // This is not going to show up on S3Guard.
|
|
|
+ Path child = new Path(firstPath, "child");
|
|
|
+ StoreContext ctx = fs.createStoreContext();
|
|
|
+ String childKey = ctx.pathToKey(child);
|
|
|
+ String baseKey = ctx.pathToKey(base) + "/";
|
|
|
+ AmazonS3 s3 = fs.getAmazonS3ClientForTesting("LIST");
|
|
|
+ String bucket = ctx.getBucket();
|
|
|
+ try {
|
|
|
+ createEmptyObject(fs, childKey);
|
|
|
+
|
|
|
+ // Do a list
|
|
|
+ ListObjectsV2Request listReq = new ListObjectsV2Request()
|
|
|
+ .withBucketName(bucket)
|
|
|
+ .withPrefix(baseKey)
|
|
|
+ .withMaxKeys(10)
|
|
|
+ .withDelimiter("/");
|
|
|
+ ListObjectsV2Result listing = s3.listObjectsV2(listReq);
|
|
|
+
|
|
|
+ // the listing has the first path as a prefix, because of the child
|
|
|
+ Assertions.assertThat(listing.getCommonPrefixes())
|
|
|
+ .describedAs("The prefixes of a LIST of %s", base)
|
|
|
+ .contains(baseKey + first + "/");
|
|
|
+
|
|
|
+ // and the last file is one of the files
|
|
|
+ Stream<String> files = listing.getObjectSummaries()
|
|
|
+ .stream()
|
|
|
+ .map(S3ObjectSummary::getKey);
|
|
|
+ Assertions.assertThat(files)
|
|
|
+ .describedAs("The files of a LIST of %s", base)
|
|
|
+ .contains(baseKey + last);
|
|
|
+
|
|
|
+ // verify absolutely that the last file exists
|
|
|
+ assertPathExists("last file", lastPath);
|
|
|
+
|
|
|
+ boolean isDDB = fs.getMetadataStore() instanceof DynamoDBMetadataStore;
|
|
|
+ // if DDB is the metastore, then we expect no FS requests to be made
|
|
|
+ // at all.
|
|
|
+ S3ATestUtils.MetricDiff listMetric = new S3ATestUtils.MetricDiff(fs,
|
|
|
+ Statistic.OBJECT_LIST_REQUESTS);
|
|
|
+ S3ATestUtils.MetricDiff getMetric = new S3ATestUtils.MetricDiff(fs,
|
|
|
+ Statistic.OBJECT_METADATA_REQUESTS);
|
|
|
+ // do a getFile status with empty dir flag
|
|
|
+ S3AFileStatus status = getStatusWithEmptyDirFlag(fs, base);
|
|
|
+ assertNonEmptyDir(status);
|
|
|
+ if (isDDB) {
|
|
|
+ listMetric.assertDiffEquals(
|
|
|
+ "FileSystem called S3 LIST rather than use DynamoDB",
|
|
|
+ 0);
|
|
|
+ getMetric.assertDiffEquals(
|
|
|
+ "FileSystem called S3 GET rather than use DynamoDB",
|
|
|
+ 0);
|
|
|
+ LOG.info("Verified that DDB directory status was accepted");
|
|
|
+ }
|
|
|
+
|
|
|
+ } finally {
|
|
|
+ // try to recover from the defective state.
|
|
|
+ s3.deleteObject(bucket, childKey);
|
|
|
+ fs.delete(lastPath, true);
|
|
|
+ ddbMs.forgetMetadata(firstPath);
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ protected void assertNonEmptyDir(final S3AFileStatus status) {
|
|
|
+ assertEquals("Should not be empty dir: " + status, Tristate.FALSE,
|
|
|
+ status.isEmptyDirectory());
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Get the DynamoDB metastore; assume false if it is of a different
|
|
|
+ * type.
|
|
|
+ * @return extracted and cast metadata store.
|
|
|
+ */
|
|
|
+ @SuppressWarnings("ConstantConditions")
|
|
|
+ private DynamoDBMetadataStore getRequiredDDBMetastore(S3AFileSystem fs) {
|
|
|
+ MetadataStore ms = fs.getMetadataStore();
|
|
|
+ assume("Not a DynamoDBMetadataStore: " + ms,
|
|
|
+ ms instanceof DynamoDBMetadataStore);
|
|
|
+ return (DynamoDBMetadataStore) ms;
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * From {@code S3AFileSystem.createEmptyObject()}.
|
|
|
+ * @param fs filesystem
|
|
|
+ * @param key key
|
|
|
+ */
|
|
|
+ private void createEmptyObject(S3AFileSystem fs, String key) {
|
|
|
+ final InputStream im = new InputStream() {
|
|
|
+ @Override
|
|
|
+ public int read() {
|
|
|
+ return -1;
|
|
|
+ }
|
|
|
+ };
|
|
|
+
|
|
|
+ PutObjectRequest putObjectRequest = fs.newPutObjectRequest(key,
|
|
|
+ fs.newObjectMetadata(0L),
|
|
|
+ im);
|
|
|
+ AmazonS3 s3 = fs.getAmazonS3ClientForTesting("PUT");
|
|
|
+ s3.putObject(putObjectRequest);
|
|
|
+ }
|
|
|
+
|
|
|
}
|