Sfoglia il codice sorgente

HDFS-10659. Namenode crashes after Journalnode re-installation in an HA cluster due to missing paxos directory. Contributed by star, Hanisha Koneru.

Signed-off-by: Wei-Chiu Chuang <weichiu@apache.org>
(cherry picked from commit 7217494f40dd99068a3f3b155261b1dac6c67828)
(cherry picked from commit 73ab4a42a650b9f8eb2c1d3e723145d70e62e781)
Hanisha Koneru 5 anni fa
parent
commit
09c108f1a0

+ 17 - 13
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/qjournal/server/JNStorage.java

@@ -81,7 +81,8 @@ class JNStorage extends Storage {
   }
 
   @Override
-  public boolean isPreUpgradableLayout(StorageDirectory sd) throws IOException {
+  public boolean isPreUpgradableLayout(StorageDirectory sd)
+      throws IOException {
     return false;
   }
 
@@ -89,7 +90,8 @@ class JNStorage extends Storage {
    * Find an edits file spanning the given transaction ID range.
    * If no such file exists, an exception is thrown.
    */
-  File findFinalizedEditsFile(long startTxId, long endTxId) throws IOException {
+  File findFinalizedEditsFile(long startTxId, long endTxId)
+      throws IOException {
     File ret = new File(sd.getCurrentDir(),
         NNStorage.getFinalizedEditsFileName(startTxId, endTxId));
     if (!ret.exists()) {
@@ -126,11 +128,18 @@ class JNStorage extends Storage {
    * paxos-like recovery process for the given log segment.
    */
   File getPaxosFile(long segmentTxId) {
-    return new File(getPaxosDir(), String.valueOf(segmentTxId));
+    return new File(getOrCreatePaxosDir(), String.valueOf(segmentTxId));
   }
   
-  File getPaxosDir() {
-    return new File(sd.getCurrentDir(), "paxos");
+  File getOrCreatePaxosDir() {
+    File paxosDir = new File(sd.getCurrentDir(), "paxos");
+    if(!paxosDir.exists()) {
+      LOG.info("Creating paxos dir: " + paxosDir.toPath());
+      if(!paxosDir.mkdir()) {
+        LOG.error("Could not create paxos dir: " + paxosDir.toPath());
+      }
+    }
+    return paxosDir;
   }
   
   File getRoot() {
@@ -144,7 +153,8 @@ class JNStorage extends Storage {
   void purgeDataOlderThan(long minTxIdToKeep) throws IOException {
     purgeMatching(sd.getCurrentDir(),
         CURRENT_DIR_PURGE_REGEXES, minTxIdToKeep);
-    purgeMatching(getPaxosDir(), PAXOS_DIR_PURGE_REGEXES, minTxIdToKeep);
+    purgeMatching(getOrCreatePaxosDir(),
+        PAXOS_DIR_PURGE_REGEXES, minTxIdToKeep);
   }
   
   /**
@@ -196,16 +206,10 @@ class JNStorage extends Storage {
     unlockAll();
     sd.clearDirectory();
     writeProperties(sd);
-    createPaxosDir();
+    getOrCreatePaxosDir();
     analyzeStorage();
   }
   
-  void createPaxosDir() throws IOException {
-    if (!getPaxosDir().mkdirs()) {
-      throw new IOException("Could not create paxos dir: " + getPaxosDir());
-    }
-  }
-  
   void analyzeStorage() throws IOException {
     this.state = sd.analyzeStorage(StartupOption.REGULAR, this);
     if (state == StorageState.NORMAL) {

+ 1 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/qjournal/server/Journal.java

@@ -1033,7 +1033,7 @@ public class Journal implements Closeable {
         + ".\n   new LV = " + storage.getLayoutVersion()
         + "; new CTime = " + storage.getCTime());
     storage.getJournalManager().doUpgrade(storage);
-    storage.createPaxosDir();
+    storage.getOrCreatePaxosDir();
     
     // Copy over the contents of the epoch data files to the new dir.
     File currentDir = storage.getSingularStorageDir().getCurrentDir();