Browse Source

HDFS-15792. ClasscastException while loading FSImage. Contributed by Renukaprasad C.

He Xiaoqiao 4 years ago
parent
commit
b54134661b

+ 5 - 4
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/AclFeature.java

@@ -19,6 +19,7 @@
 package org.apache.hadoop.hdfs.server.namenode;
 
 import java.util.Arrays;
+import java.util.concurrent.atomic.AtomicInteger;
 
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.fs.permission.AclEntry;
@@ -34,7 +35,7 @@ import org.apache.hadoop.thirdparty.com.google.common.collect.ImmutableList;
 public class AclFeature implements INode.Feature, ReferenceCounter {
   public static final ImmutableList<AclEntry> EMPTY_ENTRY_LIST =
     ImmutableList.of();
-  private int refCount = 0;
+  private AtomicInteger value = new AtomicInteger();
 
   private final int [] entries;
 
@@ -84,16 +85,16 @@ public class AclFeature implements INode.Feature, ReferenceCounter {
 
   @Override
   public int getRefCount() {
-    return refCount;
+    return value.get();
   }
 
   @Override
   public int incrementAndGetRefCount() {
-    return ++refCount;
+    return value.incrementAndGet();
   }
 
   @Override
   public int decrementAndGetRefCount() {
-    return (refCount > 0) ? --refCount : 0;
+    return value.updateAndGet(i -> i > 0 ? i - 1 : i);
   }
 }

+ 3 - 4
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/util/ReferenceCountMap.java

@@ -17,8 +17,8 @@
  */
 package org.apache.hadoop.hdfs.util;
 
-import java.util.HashMap;
 import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
 
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
@@ -37,7 +37,7 @@ import org.apache.hadoop.thirdparty.com.google.common.collect.ImmutableList;
 @InterfaceStability.Evolving
 public class ReferenceCountMap<E extends ReferenceCountMap.ReferenceCounter> {
 
-  private Map<E, E> referenceMap = new HashMap<E, E>();
+  private Map<E, E> referenceMap = new ConcurrentHashMap<>();
 
   /**
    * Add the reference. If the instance already present, just increase the
@@ -47,10 +47,9 @@ public class ReferenceCountMap<E extends ReferenceCountMap.ReferenceCounter> {
    * @return Referenced instance
    */
   public E put(E key) {
-    E value = referenceMap.get(key);
+    E value = referenceMap.putIfAbsent(key, key);
     if (value == null) {
       value = key;
-      referenceMap.put(key, value);
     }
     value.incrementAndGetRefCount();
     return value;

+ 113 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/util/TestReferenceCountMap.java

@@ -0,0 +1,113 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.hdfs.util;
+
+import org.apache.hadoop.hdfs.server.namenode.AclFeature;
+import org.junit.Assert;
+import org.junit.Test;
+
+/**
+ * Verify ReferenceCount map in concurrent scenarios
+ *
+ */
+public class TestReferenceCountMap {
+  //Add these number of references in loop
+  public static final int LOOP_COUNTER = 10000;
+  //Use 2 global features
+  AclFeature aclFeature1 = new AclFeature(new int[]{1});
+  AclFeature aclFeature2 = new AclFeature(new int[]{2});
+
+  @Test
+  public void testReferenceCountMap() throws Exception{
+    ReferenceCountMap<AclFeature> countMap = new ReferenceCountMap<>();
+    countMap.put(aclFeature1);
+    countMap.put(aclFeature2);
+    Assert.assertEquals(1, countMap.getReferenceCount(aclFeature1));
+    Assert.assertEquals(1, countMap.getReferenceCount(aclFeature2));
+
+    countMap.put(aclFeature1);
+    countMap.put(aclFeature2);
+    Assert.assertEquals(2, countMap.getReferenceCount(aclFeature1));
+    Assert.assertEquals(2, countMap.getReferenceCount(aclFeature2));
+
+    countMap.put(aclFeature1);
+    Assert.assertEquals(3, countMap.getReferenceCount(aclFeature1));
+    countMap.put(aclFeature1);
+    Assert.assertEquals(4, countMap.getReferenceCount(aclFeature1));
+    Assert.assertEquals(2, countMap.getReferenceCount(aclFeature2));
+
+    //Delete operations:
+    countMap.remove(aclFeature1);
+    countMap.remove(aclFeature2);
+    Assert.assertEquals(3, countMap.getReferenceCount(aclFeature1));
+    Assert.assertEquals(1, countMap.getReferenceCount(aclFeature2));
+
+    //Verify unique elements in map
+    Assert.assertEquals(2, countMap.getUniqueElementsSize());
+  }
+
+  @Test
+  public void testRefCountMapConcurrently() throws Exception{
+    ReferenceCountMap<AclFeature> countMap = new ReferenceCountMap<>();
+
+    PutThread putThread1 = new PutThread(countMap);
+    putThread1.start();
+    PutThread putThread2 = new PutThread(countMap);
+    putThread2.start();
+    RemoveThread removeThread1 = new RemoveThread(countMap);
+
+    putThread1.join();
+    putThread2.join();
+    Assert.assertEquals(2 * LOOP_COUNTER, countMap.getReferenceCount(aclFeature1));
+    Assert.assertEquals(2 * LOOP_COUNTER, countMap.getReferenceCount(aclFeature2));
+
+    removeThread1.start();
+    removeThread1.join();
+    Assert.assertEquals(LOOP_COUNTER, countMap.getReferenceCount(aclFeature1));
+    Assert.assertEquals(LOOP_COUNTER, countMap.getReferenceCount(aclFeature2));
+  }
+
+  class PutThread extends Thread{
+    ReferenceCountMap<AclFeature> referenceCountMap;
+    public PutThread(ReferenceCountMap<AclFeature> referenceCountMap){
+      this.referenceCountMap = referenceCountMap;
+    }
+    @Override
+    public void run() {
+      for (int i = 0; i < LOOP_COUNTER; i++) {
+        referenceCountMap.put(aclFeature1);
+        referenceCountMap.put(aclFeature2);
+      }
+    }
+  };
+
+  class RemoveThread extends Thread{
+    ReferenceCountMap<AclFeature> referenceCountMap;
+    public RemoveThread(ReferenceCountMap<AclFeature> referenceCountMap){
+      this.referenceCountMap = referenceCountMap;
+    }
+    @Override
+    public void run() {
+      for (int i = 0; i < LOOP_COUNTER; i++) {
+        referenceCountMap.remove(aclFeature1);
+        referenceCountMap.remove(aclFeature2);
+      }
+    }
+  };
+}