Browse Source

Opening of rocksDB in datanode fails with "No locks available"

Signed-off-by: Nanda kumar <nanda@apache.org>
Mukul Kumar Singh 6 years ago
parent
commit
277e9a835b

+ 5 - 9
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/utils/ContainerCache.java

@@ -77,7 +77,8 @@ public final class ContainerCache extends LRUMap {
       while (iterator.hasNext()) {
         iterator.next();
         ReferenceCountedDB db = (ReferenceCountedDB) iterator.getValue();
-        db.setEvicted(true);
+        Preconditions.checkArgument(db.cleanup(), "refCount:",
+            db.getReferenceCount());
       }
       // reset the cache
       cache.clear();
@@ -92,14 +93,9 @@ public final class ContainerCache extends LRUMap {
   @Override
   protected boolean removeLRU(LinkEntry entry) {
     ReferenceCountedDB db = (ReferenceCountedDB) entry.getValue();
-    String dbFile = (String)entry.getKey();
     lock.lock();
     try {
-      db.setEvicted(false);
-      return true;
-    } catch (Exception e) {
-      LOG.error("Eviction for db:{} failed", dbFile, e);
-      return false;
+      return db.cleanup();
     } finally {
       lock.unlock();
     }
@@ -156,8 +152,8 @@ public final class ContainerCache extends LRUMap {
     try {
       ReferenceCountedDB db = (ReferenceCountedDB)this.get(containerDBPath);
       if (db != null) {
-        // marking it as evicted will close the db as well.
-        db.setEvicted(true);
+        Preconditions.checkArgument(db.cleanup(), "refCount:",
+            db.getReferenceCount());
       }
       this.remove(containerDBPath);
     } finally {

+ 12 - 16
hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/utils/ReferenceCountedDB.java

@@ -24,7 +24,6 @@ import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
 import java.io.Closeable;
-import java.util.concurrent.atomic.AtomicBoolean;
 import java.util.concurrent.atomic.AtomicInteger;
 
 /**
@@ -38,17 +37,19 @@ public class ReferenceCountedDB implements Closeable {
   private static final Logger LOG =
       LoggerFactory.getLogger(ReferenceCountedDB.class);
   private final AtomicInteger referenceCount;
-  private final AtomicBoolean isEvicted;
   private final MetadataStore store;
   private final String containerDBPath;
 
   public ReferenceCountedDB(MetadataStore store, String containerDBPath) {
     this.referenceCount = new AtomicInteger(0);
-    this.isEvicted = new AtomicBoolean(false);
     this.store = store;
     this.containerDBPath = containerDBPath;
   }
 
+  public long getReferenceCount() {
+    return referenceCount.get();
+  }
+
   public void incrementReference() {
     this.referenceCount.incrementAndGet();
     if (LOG.isDebugEnabled()) {
@@ -59,35 +60,30 @@ public class ReferenceCountedDB implements Closeable {
   }
 
   public void decrementReference() {
-    this.referenceCount.decrementAndGet();
+    int refCount = this.referenceCount.decrementAndGet();
+    Preconditions.checkArgument(refCount >= 0, "refCount:", refCount);
     if (LOG.isDebugEnabled()) {
       LOG.debug("DecRef {} to refCnt {} \n", containerDBPath,
           referenceCount.get());
       new Exception().printStackTrace();
     }
-    cleanup();
-  }
-
-  public void setEvicted(boolean checkNoReferences) {
-    Preconditions.checkState(!checkNoReferences ||
-            (referenceCount.get() == 0),
-        "checkNoReferences:%b, referencount:%d, dbPath:%s",
-        checkNoReferences, referenceCount.get(), containerDBPath);
-    isEvicted.set(true);
-    cleanup();
   }
 
-  private void cleanup() {
-    if (referenceCount.get() == 0 && isEvicted.get() && store != null) {
+  public boolean cleanup() {
+    if (referenceCount.get() == 0 && store != null) {
       if (LOG.isDebugEnabled()) {
         LOG.debug("Close {} refCnt {}", containerDBPath,
             referenceCount.get());
       }
       try {
         store.close();
+        return true;
       } catch (Exception e) {
         LOG.error("Error closing DB. Container: " + containerDBPath, e);
+        return false;
       }
+    } else {
+      return false;
     }
   }
 

+ 128 - 0
hadoop-hdds/container-service/src/test/java/org/apache/hadoop/ozone/container/common/TestContainerCache.java

@@ -0,0 +1,128 @@
+/*
+ * 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.ozone.container.common;
+
+import org.apache.hadoop.fs.FileSystemTestHelper;
+import org.apache.hadoop.hdds.conf.OzoneConfiguration;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.container.common.utils.ContainerCache;
+import org.apache.hadoop.ozone.container.common.utils.ReferenceCountedDB;
+import org.apache.hadoop.utils.MetadataStore;
+import org.apache.hadoop.utils.MetadataStoreBuilder;
+import org.junit.Assert;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.ExpectedException;
+
+import java.io.File;
+
+
+/**
+ * Test ContainerCache with evictions.
+ */
+public class TestContainerCache {
+  private static String testRoot = new FileSystemTestHelper().getTestRootDir();
+
+  @Rule
+  public ExpectedException thrown = ExpectedException.none();
+
+  private void createContainerDB(OzoneConfiguration conf, File dbFile)
+      throws Exception {
+    MetadataStore store = MetadataStoreBuilder.newBuilder().setConf(conf)
+        .setCreateIfMissing(true).setDbFile(dbFile).build();
+
+    // we close since the SCM pre-creates containers.
+    // we will open and put Db handle into a cache when keys are being created
+    // in a container.
+
+    store.close();
+  }
+
+  @Test
+  public void testContainerCacheEviction() throws Exception {
+    File root = new File(testRoot);
+    root.mkdirs();
+
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.setInt(OzoneConfigKeys.OZONE_CONTAINER_CACHE_SIZE, 2);
+
+    ContainerCache cache = ContainerCache.getInstance(conf);
+    File containerDir1 = new File(root, "cont1");
+    File containerDir2 = new File(root, "cont2");
+    File containerDir3 = new File(root, "cont3");
+    File containerDir4 = new File(root, "cont4");
+
+
+    createContainerDB(conf, containerDir1);
+    createContainerDB(conf, containerDir2);
+    createContainerDB(conf, containerDir3);
+    createContainerDB(conf, containerDir4);
+
+    // Get 2 references out of the same db and verify the objects are same.
+    ReferenceCountedDB db1 = cache.getDB(1, "RocksDB",
+        containerDir1.getPath(), conf);
+    Assert.assertEquals(1, db1.getReferenceCount());
+    ReferenceCountedDB db2 = cache.getDB(1, "RocksDB",
+        containerDir1.getPath(), conf);
+    Assert.assertEquals(2, db2.getReferenceCount());
+    Assert.assertEquals(2, db1.getReferenceCount());
+    Assert.assertEquals(db1, db2);
+
+    // add one more references to ContainerCache.
+    ReferenceCountedDB db3 = cache.getDB(2, "RocksDB",
+        containerDir2.getPath(), conf);
+    Assert.assertEquals(1, db3.getReferenceCount());
+
+    // and close the reference
+    db3.close();
+    Assert.assertEquals(0, db3.getReferenceCount());
+
+    Assert.assertTrue(cache.isFull());
+
+    // add one more reference to ContainerCache and verify that it will not
+    // evict the least recent entry as it has reference.
+    ReferenceCountedDB db4 = cache.getDB(3, "RocksDB",
+        containerDir3.getPath(), conf);
+    Assert.assertEquals(1, db4.getReferenceCount());
+
+    Assert.assertEquals(2, cache.size());
+    Assert.assertNotNull(cache.get(containerDir1.getPath()));
+    Assert.assertNull(cache.get(containerDir2.getPath()));
+
+    // Now close both the references for container1
+    db1.close();
+    db2.close();
+    Assert.assertEquals(0, db1.getReferenceCount());
+    Assert.assertEquals(0, db2.getReferenceCount());
+
+
+    // The reference count for container1 is 0 but it is not evicted.
+    ReferenceCountedDB db5 = cache.getDB(1, "RocksDB",
+        containerDir1.getPath(), conf);
+    Assert.assertEquals(1, db5.getReferenceCount());
+    Assert.assertEquals(db1, db5);
+    db5.close();
+    db4.close();
+
+
+    // Decrementing reference count below zero should fail.
+    thrown.expect(IllegalArgumentException.class);
+    db5.close();
+  }
+}