You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@hbase.apache.org by GitBox <gi...@apache.org> on 2020/05/29 23:48:06 UTC

[GitHub] [hbase] ndimiduk commented on a change in pull request #1806: HBASE-24471 The way we bootstrap meta table is confusing

ndimiduk commented on a change in pull request #1806:
URL: https://github.com/apache/hbase/pull/1806#discussion_r432767573



##########
File path: hbase-protocol-shaded/src/main/protobuf/server/master/MasterProcedure.proto
##########
@@ -489,8 +489,9 @@ message ReopenTableRegionsStateData {
 }
 
 enum InitMetaState {
-  INIT_META_ASSIGN_META = 1;
-  INIT_META_CREATE_NAMESPACES = 2;
+  INIT_META_WRITE_FS_LAYOUT = 1;

Review comment:
       Why not preserve the existing ordinals? The state machine implemented over this enum can control the transitions however it sees fit, and you'll can keep serialized data that much more backwards-compatible.
   
   Unless these existing states have new meanings in the new paradigm...

##########
File path: hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterFileSystem.java
##########
@@ -245,61 +238,56 @@ public ClusterId getClusterId() {
    *         directory with necessary bootup files).
    */
   private Path checkRootDir(final Path rd, final Configuration c, final FileSystem fs)
-      throws IOException {
+    throws IOException {
     // If FS is in safe mode wait till out of it.
     FSUtils.waitOnSafeMode(c, c.getInt(HConstants.THREAD_WAKE_FREQUENCY, 10 * 1000));
 
     // Filesystem is good. Go ahead and check for hbase.rootdir.
+    FileStatus status;
+    try {
+      status = fs.getFileStatus(rd);
+    } catch (FileNotFoundException e) {
+      status = null;
+    }
     try {
-      if (!fs.exists(rd)) {
-        fs.mkdirs(rd);
+      if (status == null) {
+        if (!fs.mkdirs(rd)) {
+          throw new IOException("Can not create root dir: " + rd);
+        }
         // DFS leaves safe mode with 0 DNs when there are 0 blocks.
         // We used to handle this by checking the current DN count and waiting until
         // it is nonzero. With security, the check for datanode count doesn't work --
         // it is a privileged op. So instead we adopt the strategy of the jobtracker
         // and simply retry file creation during bootstrap indefinitely. As soon as
         // there is one datanode it will succeed. Permission problems should have
         // already been caught by mkdirs above.
-        FSUtils.setVersion(fs, rd, c.getInt(HConstants.THREAD_WAKE_FREQUENCY,
-          10 * 1000), c.getInt(HConstants.VERSION_FILE_WRITE_ATTEMPTS,
-            HConstants.DEFAULT_VERSION_FILE_WRITE_ATTEMPTS));
+        FSUtils.setVersion(fs, rd, c.getInt(HConstants.THREAD_WAKE_FREQUENCY, 10 * 1000), c.getInt(
+          HConstants.VERSION_FILE_WRITE_ATTEMPTS, HConstants.DEFAULT_VERSION_FILE_WRITE_ATTEMPTS));
       } else {
-        if (!fs.isDirectory(rd)) {
-          throw new IllegalArgumentException(rd.toString() + " is not a directory");
+        if (!status.isDirectory()) {
+          throw new IllegalArgumentException(rd + " is not a directory");

Review comment:
       How about `"Configured '" + HConstants.HBASE_DIR + "' + rd + " is not a directory."` ?

##########
File path: hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/InitMetaProcedure.java
##########
@@ -64,41 +72,68 @@ public TableOperationType getTableOperationType() {
     return TableOperationType.CREATE;
   }
 
+  private static void writeFsLayout(Path rootDir, Configuration conf) throws IOException {
+    LOG.info("BOOTSTRAP: creating hbase:meta region");
+    FileSystem fs = rootDir.getFileSystem(conf);
+    Path tableDir = CommonFSUtils.getTableDir(rootDir, TableName.META_TABLE_NAME);
+    if (fs.exists(tableDir) && !fs.delete(tableDir, true)) {
+      LOG.warn("Can not delete partial created meta table, continue...");
+    }
+    // Bootstrapping, make sure blockcache is off. Else, one will be
+    // created here in bootstrap and it'll need to be cleaned up. Better to
+    // not make it in first place. Turn off block caching for bootstrap.
+    // Enable after.
+    FSTableDescriptors.tryUpdateMetaTableDescriptor(conf, fs, rootDir,
+      builder -> builder.setRegionReplication(
+        conf.getInt(HConstants.META_REPLICAS_NUM, HConstants.DEFAULT_META_REPLICA_NUM)));
+    TableDescriptor metaDescriptor = new FSTableDescriptors(conf).get(TableName.META_TABLE_NAME);
+    HRegion
+      .createHRegion(RegionInfoBuilder.FIRST_META_REGIONINFO, rootDir, conf, metaDescriptor, null)
+      .close();
+  }
+
   @Override
   protected Flow executeFromState(MasterProcedureEnv env, InitMetaState state)
-      throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
+    throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
     LOG.debug("Execute {}", this);
-    switch (state) {
-      case INIT_META_ASSIGN_META:
-        LOG.info("Going to assign meta");
-        addChildProcedure(env.getAssignmentManager()
-          .createAssignProcedures(Arrays.asList(RegionInfoBuilder.FIRST_META_REGIONINFO)));
-        setNextState(InitMetaState.INIT_META_CREATE_NAMESPACES);
-        return Flow.HAS_MORE_STATE;
-      case INIT_META_CREATE_NAMESPACES:
-        LOG.info("Going to create {} and {} namespaces", DEFAULT_NAMESPACE, SYSTEM_NAMESPACE);
-        try {
+    try {
+      switch (state) {
+        case INIT_META_WRITE_FS_LAYOUT:
+          Configuration conf = env.getMasterConfiguration();
+          Path rootDir = CommonFSUtils.getRootDir(conf);
+          writeFsLayout(rootDir, conf);
+          setNextState(InitMetaState.INIT_META_ASSIGN_META);
+          return Flow.HAS_MORE_STATE;
+        case INIT_META_ASSIGN_META:
+          LOG.info("Going to assign meta");
+          addChildProcedure(env.getAssignmentManager()

Review comment:
       Oh, it is because you use `skipPersistence()` down below? The `ProcedureExecutor` and `ProcedureStore` can function without the root filesystem being available?

##########
File path: hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/InitMetaProcedure.java
##########
@@ -64,41 +72,68 @@ public TableOperationType getTableOperationType() {
     return TableOperationType.CREATE;
   }
 
+  private static void writeFsLayout(Path rootDir, Configuration conf) throws IOException {
+    LOG.info("BOOTSTRAP: creating hbase:meta region");
+    FileSystem fs = rootDir.getFileSystem(conf);
+    Path tableDir = CommonFSUtils.getTableDir(rootDir, TableName.META_TABLE_NAME);
+    if (fs.exists(tableDir) && !fs.delete(tableDir, true)) {
+      LOG.warn("Can not delete partial created meta table, continue...");
+    }
+    // Bootstrapping, make sure blockcache is off. Else, one will be
+    // created here in bootstrap and it'll need to be cleaned up. Better to
+    // not make it in first place. Turn off block caching for bootstrap.
+    // Enable after.
+    FSTableDescriptors.tryUpdateMetaTableDescriptor(conf, fs, rootDir,
+      builder -> builder.setRegionReplication(
+        conf.getInt(HConstants.META_REPLICAS_NUM, HConstants.DEFAULT_META_REPLICA_NUM)));
+    TableDescriptor metaDescriptor = new FSTableDescriptors(conf).get(TableName.META_TABLE_NAME);
+    HRegion
+      .createHRegion(RegionInfoBuilder.FIRST_META_REGIONINFO, rootDir, conf, metaDescriptor, null)
+      .close();
+  }
+
   @Override
   protected Flow executeFromState(MasterProcedureEnv env, InitMetaState state)
-      throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
+    throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
     LOG.debug("Execute {}", this);
-    switch (state) {
-      case INIT_META_ASSIGN_META:
-        LOG.info("Going to assign meta");
-        addChildProcedure(env.getAssignmentManager()
-          .createAssignProcedures(Arrays.asList(RegionInfoBuilder.FIRST_META_REGIONINFO)));
-        setNextState(InitMetaState.INIT_META_CREATE_NAMESPACES);
-        return Flow.HAS_MORE_STATE;
-      case INIT_META_CREATE_NAMESPACES:
-        LOG.info("Going to create {} and {} namespaces", DEFAULT_NAMESPACE, SYSTEM_NAMESPACE);
-        try {
+    try {
+      switch (state) {
+        case INIT_META_WRITE_FS_LAYOUT:
+          Configuration conf = env.getMasterConfiguration();
+          Path rootDir = CommonFSUtils.getRootDir(conf);
+          writeFsLayout(rootDir, conf);
+          setNextState(InitMetaState.INIT_META_ASSIGN_META);
+          return Flow.HAS_MORE_STATE;
+        case INIT_META_ASSIGN_META:
+          LOG.info("Going to assign meta");
+          addChildProcedure(env.getAssignmentManager()

Review comment:
       The procedures are stored in the `LocalStore`/`LocalRegion`. The `LocalRegion` is backed by `${hbase.rootdir}/MasterData`. How can we have a procedure that initializes `hbase.rootdir`? Isn't this a circular dependency?

##########
File path: hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterFileSystem.java
##########
@@ -245,61 +238,56 @@ public ClusterId getClusterId() {
    *         directory with necessary bootup files).
    */
   private Path checkRootDir(final Path rd, final Configuration c, final FileSystem fs)
-      throws IOException {
+    throws IOException {
     // If FS is in safe mode wait till out of it.
     FSUtils.waitOnSafeMode(c, c.getInt(HConstants.THREAD_WAKE_FREQUENCY, 10 * 1000));
 
     // Filesystem is good. Go ahead and check for hbase.rootdir.
+    FileStatus status;
+    try {
+      status = fs.getFileStatus(rd);
+    } catch (FileNotFoundException e) {
+      status = null;
+    }
     try {
-      if (!fs.exists(rd)) {
-        fs.mkdirs(rd);
+      if (status == null) {
+        if (!fs.mkdirs(rd)) {
+          throw new IOException("Can not create root dir: " + rd);
+        }
         // DFS leaves safe mode with 0 DNs when there are 0 blocks.
         // We used to handle this by checking the current DN count and waiting until
         // it is nonzero. With security, the check for datanode count doesn't work --
         // it is a privileged op. So instead we adopt the strategy of the jobtracker
         // and simply retry file creation during bootstrap indefinitely. As soon as
         // there is one datanode it will succeed. Permission problems should have
         // already been caught by mkdirs above.
-        FSUtils.setVersion(fs, rd, c.getInt(HConstants.THREAD_WAKE_FREQUENCY,
-          10 * 1000), c.getInt(HConstants.VERSION_FILE_WRITE_ATTEMPTS,
-            HConstants.DEFAULT_VERSION_FILE_WRITE_ATTEMPTS));
+        FSUtils.setVersion(fs, rd, c.getInt(HConstants.THREAD_WAKE_FREQUENCY, 10 * 1000), c.getInt(

Review comment:
       I know it's not your code, but mind unpacking these inline configuration lookups? They're terribly illegible.

##########
File path: hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterFileSystem.java
##########
@@ -245,61 +238,56 @@ public ClusterId getClusterId() {
    *         directory with necessary bootup files).
    */
   private Path checkRootDir(final Path rd, final Configuration c, final FileSystem fs)

Review comment:
       Indeed, static analysis tells me that this method's return value is never used. Perhaps this is the legacy at some attempt at a fluent style?




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org