|
@@ -19,32 +19,35 @@ package org.apache.hadoop.hdfs.server.namenode;
|
|
|
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.fs.Path;
|
|
|
+import org.apache.hadoop.fs.XAttr;
|
|
|
+import org.apache.hadoop.fs.XAttrSetFlag;
|
|
|
+import org.apache.hadoop.hdfs.DFSConfigKeys;
|
|
|
import org.apache.hadoop.hdfs.DistributedFileSystem;
|
|
|
import org.apache.hadoop.hdfs.MiniDFSCluster;
|
|
|
-import org.apache.hadoop.hdfs.protocol.SnapshotException;
|
|
|
+import org.apache.hadoop.hdfs.XAttrHelper;
|
|
|
+import org.apache.hadoop.hdfs.protocol.HdfsConstants;
|
|
|
+import org.apache.hadoop.hdfs.server.common.HdfsServerConstants;
|
|
|
import org.apache.hadoop.hdfs.server.namenode.snapshot.SnapshotTestHelper;
|
|
|
-import org.apache.hadoop.test.GenericTestUtils;
|
|
|
import org.junit.After;
|
|
|
-import org.junit.Assert;
|
|
|
import org.junit.Before;
|
|
|
import org.junit.Test;
|
|
|
-import org.slf4j.Logger;
|
|
|
-import org.slf4j.LoggerFactory;
|
|
|
-import org.slf4j.event.Level;
|
|
|
|
|
|
import java.io.IOException;
|
|
|
+import java.util.Arrays;
|
|
|
+import java.util.EnumSet;
|
|
|
+import java.util.Map;
|
|
|
|
|
|
-import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_SNAPSHOT_DELETION_ORDERED;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.
|
|
|
+ DFS_NAMENODE_SNAPSHOT_DELETION_ORDERED;
|
|
|
+import static org.junit.Assert.assertNull;
|
|
|
+import static org.junit.Assert.assertTrue;
|
|
|
|
|
|
-/** Test ordered snapshot deletion. */
|
|
|
+/**
|
|
|
+ * Test ordered snapshot deletion.
|
|
|
+ */
|
|
|
public class TestOrderedSnapshotDeletion {
|
|
|
- static final Logger LOG = LoggerFactory.getLogger(FSDirectory.class);
|
|
|
-
|
|
|
- {
|
|
|
- SnapshotTestHelper.disableLogs();
|
|
|
- GenericTestUtils.setLogLevel(INode.LOG, Level.TRACE);
|
|
|
- }
|
|
|
-
|
|
|
+ static final String xattrName = "user.a1";
|
|
|
+ static final byte[] xattrValue = {0x31, 0x32, 0x33};
|
|
|
private final Path snapshottableDir
|
|
|
= new Path("/" + getClass().getSimpleName());
|
|
|
|
|
@@ -67,7 +70,7 @@ public class TestOrderedSnapshotDeletion {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- @Test (timeout=60000)
|
|
|
+ @Test(timeout = 60000)
|
|
|
public void testConf() throws Exception {
|
|
|
DistributedFileSystem hdfs = cluster.getFileSystem();
|
|
|
hdfs.mkdirs(snapshottableDir);
|
|
@@ -85,21 +88,125 @@ public class TestOrderedSnapshotDeletion {
|
|
|
hdfs.mkdirs(sub2);
|
|
|
hdfs.createSnapshot(snapshottableDir, "s2");
|
|
|
|
|
|
- assertDeletionDenied(snapshottableDir, "s1", hdfs);
|
|
|
- assertDeletionDenied(snapshottableDir, "s2", hdfs);
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ assertXAttrSet("s2", hdfs, null);
|
|
|
hdfs.deleteSnapshot(snapshottableDir, "s0");
|
|
|
- assertDeletionDenied(snapshottableDir, "s2", hdfs);
|
|
|
+ assertXAttrSet("s2", hdfs, null);
|
|
|
hdfs.deleteSnapshot(snapshottableDir, "s1");
|
|
|
hdfs.deleteSnapshot(snapshottableDir, "s2");
|
|
|
}
|
|
|
|
|
|
- static void assertDeletionDenied(Path snapshottableDir, String snapshot,
|
|
|
- DistributedFileSystem hdfs) throws IOException {
|
|
|
+ void assertXAttrSet(String snapshot,
|
|
|
+ DistributedFileSystem hdfs, XAttr newXattr)
|
|
|
+ throws IOException {
|
|
|
+ hdfs.deleteSnapshot(snapshottableDir, snapshot);
|
|
|
+ // Check xAttr for parent directory
|
|
|
+ FSNamesystem namesystem = cluster.getNamesystem();
|
|
|
+ Path snapshotRoot = SnapshotTestHelper.getSnapshotRoot(snapshottableDir,
|
|
|
+ snapshot);
|
|
|
+ INode inode = namesystem.getFSDirectory().getINode(snapshotRoot.toString());
|
|
|
+ XAttrFeature f = inode.getXAttrFeature();
|
|
|
+ XAttr xAttr = f.getXAttr(HdfsServerConstants.SNAPSHOT_XATTR_NAME);
|
|
|
+ assertTrue("Snapshot xAttr should exist", xAttr != null);
|
|
|
+ assertTrue(xAttr.getName().equals(HdfsServerConstants.SNAPSHOT_XATTR_NAME.
|
|
|
+ replace("system.", "")));
|
|
|
+ assertTrue(xAttr.getNameSpace().equals(XAttr.NameSpace.SYSTEM));
|
|
|
+ assertNull(xAttr.getValue());
|
|
|
+
|
|
|
+ // Make sure its not user visible
|
|
|
+ if (cluster.getNameNode().getConf().getBoolean(DFSConfigKeys.
|
|
|
+ DFS_NAMENODE_XATTRS_ENABLED_KEY,
|
|
|
+ DFSConfigKeys.DFS_NAMENODE_XATTRS_ENABLED_DEFAULT)) {
|
|
|
+ Map<String, byte[]> xattrMap = hdfs.getXAttrs(snapshotRoot);
|
|
|
+ assertTrue(newXattr == null ? xattrMap.isEmpty() :
|
|
|
+ Arrays.equals(newXattr.getValue(), xattrMap.get(xattrName)));
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test(timeout = 60000)
|
|
|
+ public void testSnapshotXattrPersistence() throws Exception {
|
|
|
+ DistributedFileSystem hdfs = cluster.getFileSystem();
|
|
|
+ hdfs.mkdirs(snapshottableDir);
|
|
|
+ hdfs.allowSnapshot(snapshottableDir);
|
|
|
+
|
|
|
+ final Path sub0 = new Path(snapshottableDir, "sub0");
|
|
|
+ hdfs.mkdirs(sub0);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s0");
|
|
|
+
|
|
|
+ final Path sub1 = new Path(snapshottableDir, "sub1");
|
|
|
+ hdfs.mkdirs(sub1);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s1");
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ cluster.restartNameNodes();
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test(timeout = 60000)
|
|
|
+ public void testSnapshotXattrWithSaveNameSpace() throws Exception {
|
|
|
+ DistributedFileSystem hdfs = cluster.getFileSystem();
|
|
|
+ hdfs.mkdirs(snapshottableDir);
|
|
|
+ hdfs.allowSnapshot(snapshottableDir);
|
|
|
+
|
|
|
+ final Path sub0 = new Path(snapshottableDir, "sub0");
|
|
|
+ hdfs.mkdirs(sub0);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s0");
|
|
|
+
|
|
|
+ final Path sub1 = new Path(snapshottableDir, "sub1");
|
|
|
+ hdfs.mkdirs(sub1);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s1");
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ hdfs.setSafeMode(HdfsConstants.SafeModeAction.SAFEMODE_ENTER);
|
|
|
+ hdfs.saveNamespace();
|
|
|
+ hdfs.setSafeMode(HdfsConstants.SafeModeAction.SAFEMODE_LEAVE);
|
|
|
+ cluster.restartNameNodes();
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test(timeout = 60000)
|
|
|
+ public void testSnapshotXattrWithDisablingXattr() throws Exception {
|
|
|
+ DistributedFileSystem hdfs = cluster.getFileSystem();
|
|
|
+ hdfs.mkdirs(snapshottableDir);
|
|
|
+ hdfs.allowSnapshot(snapshottableDir);
|
|
|
+
|
|
|
+ final Path sub0 = new Path(snapshottableDir, "sub0");
|
|
|
+ hdfs.mkdirs(sub0);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s0");
|
|
|
+
|
|
|
+ final Path sub1 = new Path(snapshottableDir, "sub1");
|
|
|
+ hdfs.mkdirs(sub1);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s1");
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ cluster.getNameNode().getConf().setBoolean(
|
|
|
+ DFSConfigKeys.DFS_NAMENODE_XATTRS_ENABLED_KEY, false);
|
|
|
+ cluster.restartNameNodes();
|
|
|
+ // ensure xAttr feature is disabled
|
|
|
try {
|
|
|
- hdfs.deleteSnapshot(snapshottableDir, snapshot);
|
|
|
- Assert.fail("deleting " +snapshot + " should fail");
|
|
|
- } catch (SnapshotException se) {
|
|
|
- LOG.info("Good, it is expected to have " + se);
|
|
|
+ hdfs.getXAttrs(snapshottableDir);
|
|
|
+ } catch (Exception e) {
|
|
|
+ assertTrue(e.getMessage().contains("The XAttr operation has been " +
|
|
|
+ "rejected. Support for XAttrs has been disabled by " +
|
|
|
+ "setting dfs.namenode.xattrs.enabled to false"));
|
|
|
}
|
|
|
+ // try deleting snapshot and verify it still sets the snapshot XAttr
|
|
|
+ assertXAttrSet("s1", hdfs, null);
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test(timeout = 60000)
|
|
|
+ public void testSnapshotXAttrWithPreExistingXattrs() throws Exception {
|
|
|
+ DistributedFileSystem hdfs = cluster.getFileSystem();
|
|
|
+ hdfs.mkdirs(snapshottableDir);
|
|
|
+ hdfs.allowSnapshot(snapshottableDir);
|
|
|
+ hdfs.setXAttr(snapshottableDir, xattrName, xattrValue,
|
|
|
+ EnumSet.of(XAttrSetFlag.CREATE));
|
|
|
+ XAttr newXAttr = XAttrHelper.buildXAttr(xattrName, xattrValue);
|
|
|
+ final Path sub0 = new Path(snapshottableDir, "sub0");
|
|
|
+ hdfs.mkdirs(sub0);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s0");
|
|
|
+
|
|
|
+ final Path sub1 = new Path(snapshottableDir, "sub1");
|
|
|
+ hdfs.mkdirs(sub1);
|
|
|
+ hdfs.createSnapshot(snapshottableDir, "s1");
|
|
|
+ assertXAttrSet("s1", hdfs, newXAttr);
|
|
|
}
|
|
|
}
|