This is an automated email from the ASF dual-hosted git repository.

ddanielr pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/accumulo.git


The following commit(s) were added to refs/heads/main by this push:
     new 3f81bc32e6 Caches non-existence of ZK node for ZooCache.getChildren 
(#5143)
3f81bc32e6 is described below

commit 3f81bc32e684382fa0e36440ec24db422c4056a9
Author: Keith Turner <ktur...@apache.org>
AuthorDate: Mon Dec 9 07:31:06 2024 -0500

    Caches non-existence of ZK node for ZooCache.getChildren (#5143)
    
    * Caches non-existence of ZK node for ZooCache.getChildren
    
    For a nodes data in zookeeper ZooCache was caching when a node did not
    exist.  However for getChildren when a node did not exist it would not
    cache this and would always go to zookeeper. Also ZooCache was handling
    a null return from Zookeeper.getChildren that would never happen.  Fixed
    these issues.
    
    fixes #5047
    
    * expire cache entries and remove their watches
    
    * Update 
test/src/main/java/org/apache/accumulo/test/zookeeper/ZooCacheIT.java
    
    Co-authored-by: Christopher Tubbs <ctubb...@apache.org>
    
    * code review update
    
    * added test for watch removal and fixed bug with watch removal
    
    * Removes logging added for testing
    
    ---------
    
    Co-authored-by: Christopher Tubbs <ctubb...@apache.org>
---
 .../accumulo/core/fate/zookeeper/ZcNode.java       | 142 +++++++++++++++
 .../accumulo/core/fate/zookeeper/ZooCache.java     | 158 ++++++++---------
 .../apache/accumulo/core/util/cache/Caches.java    |   3 +-
 .../accumulo/core/fate/zookeeper/ZooCacheTest.java | 125 ++++++++++++-
 .../apache/accumulo/test/zookeeper/ZooCacheIT.java | 195 +++++++++++++++++++++
 5 files changed, 532 insertions(+), 91 deletions(-)

diff --git 
a/core/src/main/java/org/apache/accumulo/core/fate/zookeeper/ZcNode.java 
b/core/src/main/java/org/apache/accumulo/core/fate/zookeeper/ZcNode.java
new file mode 100644
index 0000000000..bcd2b93872
--- /dev/null
+++ b/core/src/main/java/org/apache/accumulo/core/fate/zookeeper/ZcNode.java
@@ -0,0 +1,142 @@
+/*
+ * 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
+ *
+ *   https://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.accumulo.core.fate.zookeeper;
+
+import java.util.List;
+import java.util.Objects;
+
+import com.google.common.base.Preconditions;
+
+/**
+ * Immutable data class used by zoo cache to hold what it is caching for 
single zookeeper node. Data
+ * and children are obtained from zookeeper at different times. This class is 
structured so that
+ * data can be obtained first and then children added later or visa veras.
+ *
+ * <p>
+ * Four distinct states can be cached for a zookeeper node.
+ * <ul>
+ * <li>Can cache that a node does not exist in zookeeper. This state is 
represented by data, state,
+ * and children all being null.</li>
+ * <li>Can cache only the data for a zookeeper node. For this state data and 
stat are non-null while
+ * children is null. Calling getChildren on node in this state will throw an 
exception.</li>
+ * <li>Can cache only the children for a zookeeper node. For this state 
children is non-null while
+ * data and stat are null. Calling getData or getStat on node in this state 
will throw an
+ * exception.</li>
+ * <li>Can cache the children and data for a zookeeper node. For this state 
data,stat, and children
+ * are non-null.</li>
+ * </ul>
+ * <p>
+ *
+ */
+class ZcNode {
+
+  private final byte[] data;
+  private final ZooCache.ZcStat stat;
+  private final List<String> children;
+
+  static final ZcNode NON_EXISTENT = new ZcNode();
+
+  private ZcNode() {
+    this.data = null;
+    this.stat = null;
+    this.children = null;
+  }
+
+  /**
+   * Creates a new ZcNode that combines the data and stat from an existing 
ZcNode and sets the
+   * children.
+   */
+  ZcNode(List<String> children, ZcNode existing) {
+    Objects.requireNonNull(children);
+    if (existing == null) {
+      this.data = null;
+      this.stat = null;
+    } else {
+      this.data = existing.data;
+      this.stat = existing.stat;
+    }
+
+    this.children = List.copyOf(children);
+  }
+
+  /**
+   * Creates a new ZcNode that combines the children from an existing ZcNode 
and sets the data and
+   * stat.
+   */
+  ZcNode(byte[] data, ZooCache.ZcStat zstat, ZcNode existing) {
+    this.data = Objects.requireNonNull(data);
+    this.stat = Objects.requireNonNull(zstat);
+    if (existing == null) {
+      this.children = null;
+    } else {
+      this.children = existing.children;
+    }
+  }
+
+  /**
+   * @return the data if the node exists and the data was set OR return null 
when the node does not
+   *         exist
+   * @throws IllegalStateException in the case where the node exists and the 
data was never set
+   */
+  byte[] getData() {
+    Preconditions.checkState(cachedData());
+    return data;
+  }
+
+  /**
+   * @return the stat if the node exists and the stat was set OR return null 
when the node does not
+   *         exist
+   * @throws IllegalStateException in the case where the node exists and the 
data was never set
+   */
+  ZooCache.ZcStat getStat() {
+    Preconditions.checkState(cachedData());
+    return stat;
+  }
+
+  /**
+   * @return the children if the node exists and the children were set OR 
return null when the node
+   *         does not exist exists
+   * @throws IllegalStateException in the case where the node exists and the 
children were never set
+   */
+  List<String> getChildren() {
+    Preconditions.checkState(cachedChildren());
+    return children;
+  }
+
+  /**
+   * @return true if the node does not exists or it exists and children are 
cached.
+   */
+  boolean cachedChildren() {
+    return children != null || notExists();
+  }
+
+  /**
+   * @return true if the node does not exists or it exists and data and stat 
cached.
+   */
+  boolean cachedData() {
+    return data != null || notExists();
+  }
+
+  /**
+   * @return true if the node does not exists in zookeeper
+   */
+  boolean notExists() {
+    return stat == null && data == null && children == null;
+  }
+}
diff --git 
a/core/src/main/java/org/apache/accumulo/core/fate/zookeeper/ZooCache.java 
b/core/src/main/java/org/apache/accumulo/core/fate/zookeeper/ZooCache.java
index 86b869fa15..ae8e752b20 100644
--- a/core/src/main/java/org/apache/accumulo/core/fate/zookeeper/ZooCache.java
+++ b/core/src/main/java/org/apache/accumulo/core/fate/zookeeper/ZooCache.java
@@ -21,10 +21,11 @@ package org.apache.accumulo.core.fate.zookeeper;
 import static java.nio.charset.StandardCharsets.UTF_8;
 import static org.apache.accumulo.core.util.LazySingletons.RANDOM;
 
+import java.time.Duration;
 import java.util.ConcurrentModificationException;
 import java.util.List;
 import java.util.Optional;
-import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
 import java.util.concurrent.atomic.AtomicLong;
 import java.util.concurrent.locks.LockSupport;
 import java.util.function.Predicate;
@@ -32,6 +33,7 @@ import java.util.function.Predicate;
 import org.apache.accumulo.core.lock.ServiceLock;
 import org.apache.accumulo.core.lock.ServiceLockData;
 import org.apache.accumulo.core.lock.ServiceLockPaths.ServiceLockPath;
+import org.apache.accumulo.core.util.cache.Caches;
 import org.apache.zookeeper.KeeperException;
 import org.apache.zookeeper.KeeperException.Code;
 import org.apache.zookeeper.WatchedEvent;
@@ -41,6 +43,8 @@ import org.apache.zookeeper.data.Stat;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
+import com.github.benmanes.caffeine.cache.Cache;
+import com.github.benmanes.caffeine.cache.RemovalListener;
 import com.google.common.annotations.VisibleForTesting;
 import com.google.common.base.Preconditions;
 
@@ -56,49 +60,10 @@ public class ZooCache {
   private static final AtomicLong nextCacheId = new AtomicLong(0);
   private final String cacheId = "ZC" + nextCacheId.incrementAndGet();
 
-  private static class ZcNode {
-    final byte[] data;
-    final ZcStat stat;
-    final boolean dataSet;
-    final List<String> children;
-    final boolean childrenSet;
-
-    private ZcNode(ZcNode other, List<String> children) {
-      this.data = other != null ? other.data : null;
-      this.stat = other != null ? other.stat : null;
-      this.dataSet = other != null ? other.dataSet : false;
-      this.children = children;
-      this.childrenSet = true;
-    }
-
-    public ZcNode(byte[] data, ZcStat zstat, ZcNode zcn) {
-      this.data = data;
-      this.stat = zstat;
-      this.dataSet = true;
-      this.children = zcn != null ? zcn.children : null;
-      this.childrenSet = zcn != null ? zcn.childrenSet : false;
-    }
-
-    byte[] getData() {
-      Preconditions.checkState(dataSet);
-      return data;
-    }
-
-    ZcStat getStat() {
-      Preconditions.checkState(dataSet);
-      return stat;
-    }
-
-    List<String> getChildren() {
-      Preconditions.checkState(childrenSet);
-      return children;
-    }
-  }
-
-  // ConcurrentHashMap will only allow one thread to run at a time for a given 
key and this
-  // implementation relies on that. Not all concurrent map implementations 
have this behavior for
+  // The concurrent map returned by Caffiene will only allow one thread to run 
at a time for a given
+  // key and ZooCache relies on that. Not all concurrent map implementations 
have this behavior for
   // their compute functions.
-  private final ConcurrentHashMap<String,ZcNode> nodeCache;
+  private final ConcurrentMap<String,ZcNode> nodeCache;
 
   private final ZooReader zReader;
 
@@ -161,6 +126,8 @@ public class ZooCache {
         case NodeChildrenChanged:
         case NodeCreated:
         case NodeDeleted:
+        case ChildWatchRemoved:
+        case DataWatchRemoved:
           remove(event.getPath());
           break;
         case None:
@@ -206,9 +173,30 @@ public class ZooCache {
    * @param watcher watcher object
    */
   public ZooCache(ZooReader reader, Watcher watcher) {
+    this(reader, watcher, Duration.ofMinutes(3));
+  }
+
+  public ZooCache(ZooReader reader, Watcher watcher, Duration timeout) {
     this.zReader = reader;
-    nodeCache = new ConcurrentHashMap<>();
     this.externalWatcher = watcher;
+    RemovalListener<String,ZcNode> removalListerner = (path, zcNode, reason) 
-> {
+      try {
+        log.trace("{} removing watches for {} because {}", cacheId, path, 
reason);
+        reader.getZooKeeper().removeWatches(path, ZooCache.this.watcher, 
Watcher.WatcherType.Any,
+            false);
+      } catch (InterruptedException | KeeperException | RuntimeException e) {
+        log.warn("{} failed to remove watches on path {} in zookeeper", 
cacheId, path, e);
+      }
+    };
+    // Must register the removal listener using evictionListener inorder for 
removal to be mutually
+    // exclusive with any other operations on the same path. This is important 
for watcher
+    // consistency, concurrently adding and removing watches for the same path 
would leave zoocache
+    // in a really bad state. The cache builder has another way to register a 
removal listener that
+    // is not mutually exclusive.
+    Cache<String,ZcNode> cache =
+        Caches.getInstance().createNewBuilder(Caches.CacheName.ZOO_CACHE, 
false)
+            
.expireAfterAccess(timeout).evictionListener(removalListerner).build();
+    nodeCache = cache.asMap();
     log.trace("{} created new cache", cacheId, new Exception());
   }
 
@@ -316,43 +304,46 @@ public class ZooCache {
       public List<String> run() throws KeeperException, InterruptedException {
 
         var zcNode = nodeCache.get(zPath);
-        if (zcNode != null && zcNode.childrenSet) {
+        if (zcNode != null && zcNode.cachedChildren()) {
           return zcNode.getChildren();
         }
 
         log.trace("{} {} was not in children cache, looking up in zookeeper", 
cacheId, zPath);
 
-        try {
-          zcNode = nodeCache.compute(zPath, (zp, zcn) -> {
-            // recheck the children now that lock is held on key
-            if (zcn != null && zcn.childrenSet) {
-              return zcn;
-            }
+        zcNode = nodeCache.compute(zPath, (zp, zcn) -> {
+          // recheck the children now that lock is held on key
+          if (zcn != null && zcn.cachedChildren()) {
+            return zcn;
+          }
 
-            try {
-              final ZooKeeper zooKeeper = getZooKeeper();
-              List<String> children;
-              children = zooKeeper.getChildren(zPath, watcher);
-              if (children != null) {
-                children = List.copyOf(children);
-              }
-              return new ZcNode(zcn, children);
-            } catch (KeeperException e) {
-              throw new ZcException(e);
-            } catch (InterruptedException e) {
-              throw new ZcInterruptedException(e);
+          try {
+            final ZooKeeper zooKeeper = getZooKeeper();
+            // Register a watcher on the node to monitor creation/deletion 
events for the node. It
+            // is possible that an event from this watch could trigger prior 
to calling getChildren.
+            // That is ok because the compute() call on the map has a lock and 
processing the event
+            // will block until compute() returns. After compute() returns the 
event processing
+            // would clear the map entry.
+            Stat stat = zooKeeper.exists(zPath, watcher);
+            if (stat == null) {
+              log.trace("{} getChildren saw that {} does not exists", cacheId, 
zPath);
+              return ZcNode.NON_EXISTENT;
             }
-          });
-          // increment this after compute call completes when the change is 
visible
-          updateCount.incrementAndGet();
-          return zcNode.getChildren();
-        } catch (ZcException zce) {
-          if (zce.getZKException().code() == Code.NONODE) {
-            return null;
-          } else {
-            throw zce;
+            List<String> children = zooKeeper.getChildren(zPath, watcher);
+            log.trace("{} adding {} children of {} to cache", cacheId, 
children.size(), zPath);
+            return new ZcNode(children, zcn);
+          } catch (KeeperException.NoNodeException nne) {
+            log.trace("{} get children saw race condition for {}, node deleted 
after exists call",
+                cacheId, zPath);
+            throw new ConcurrentModificationException(nne);
+          } catch (KeeperException e) {
+            throw new ZcException(e);
+          } catch (InterruptedException e) {
+            throw new ZcInterruptedException(e);
           }
-        }
+        });
+        // increment this after compute call completes when the change is 
visible
+        updateCount.incrementAndGet();
+        return zcNode.getChildren();
       }
     };
 
@@ -386,7 +377,7 @@ public class ZooCache {
       public byte[] run() throws KeeperException, InterruptedException {
 
         var zcNode = nodeCache.get(zPath);
-        if (zcNode != null && zcNode.dataSet) {
+        if (zcNode != null && zcNode.cachedData()) {
           if (status != null) {
             copyStats(status, zcNode.getStat());
           }
@@ -398,7 +389,7 @@ public class ZooCache {
         zcNode = nodeCache.compute(zPath, (zp, zcn) -> {
           // recheck the now that lock is held on key, it may be present now. 
Could have been
           // computed while waiting for lock.
-          if (zcn != null && zcn.dataSet) {
+          if (zcn != null && zcn.cachedData()) {
             return zcn;
           }
           /*
@@ -412,18 +403,19 @@ public class ZooCache {
           try {
             final ZooKeeper zooKeeper = getZooKeeper();
             Stat stat = zooKeeper.exists(zPath, watcher);
-            byte[] data = null;
-            ZcStat zstat = null;
             if (stat == null) {
               if (log.isTraceEnabled()) {
                 log.trace("{} zookeeper did not contain {}", cacheId, zPath);
               }
+              return ZcNode.NON_EXISTENT;
             } else {
+              byte[] data = null;
+              ZcStat zstat = null;
               try {
                 data = zooKeeper.getData(zPath, watcher, stat);
                 zstat = new ZcStat(stat);
               } catch (KeeperException.BadVersionException | 
KeeperException.NoNodeException e1) {
-                throw new ConcurrentModificationException();
+                throw new ConcurrentModificationException(e1);
               } catch (InterruptedException e) {
                 throw new ZcInterruptedException(e);
               }
@@ -431,8 +423,8 @@ public class ZooCache {
                 log.trace("{} zookeeper contained {} {}", cacheId, zPath,
                     (data == null ? null : new String(data, UTF_8)));
               }
+              return new ZcNode(data, zstat, zcn);
             }
-            return new ZcNode(data, zstat, zcn);
           } catch (KeeperException ke) {
             throw new ZcException(ke);
           } catch (InterruptedException e) {
@@ -502,9 +494,9 @@ public class ZooCache {
    * @return true if data value is cached
    */
   @VisibleForTesting
-  boolean dataCached(String zPath) {
+  public boolean dataCached(String zPath) {
     var zcn = nodeCache.get(zPath);
-    return zcn != null && zcn.dataSet;
+    return zcn != null && zcn.cachedData();
   }
 
   /**
@@ -514,9 +506,9 @@ public class ZooCache {
    * @return true if children are cached
    */
   @VisibleForTesting
-  boolean childrenCached(String zPath) {
+  public boolean childrenCached(String zPath) {
     var zcn = nodeCache.get(zPath);
-    return zcn != null && zcn.childrenSet;
+    return zcn != null && zcn.cachedChildren();
   }
 
   /**
diff --git a/core/src/main/java/org/apache/accumulo/core/util/cache/Caches.java 
b/core/src/main/java/org/apache/accumulo/core/util/cache/Caches.java
index 67dc8f5bb5..8724f87e04 100644
--- a/core/src/main/java/org/apache/accumulo/core/util/cache/Caches.java
+++ b/core/src/main/java/org/apache/accumulo/core/util/cache/Caches.java
@@ -65,7 +65,8 @@ public class Caches implements MetricsProducer {
     TSRM_FILE_LENGTHS,
     TINYLFU_BLOCK_CACHE,
     VOLUME_HDFS_CONFIGS,
-    MINC_AGE
+    MINC_AGE,
+    ZOO_CACHE
   }
 
   private static final Logger LOG = LoggerFactory.getLogger(Caches.class);
diff --git 
a/core/src/test/java/org/apache/accumulo/core/fate/zookeeper/ZooCacheTest.java 
b/core/src/test/java/org/apache/accumulo/core/fate/zookeeper/ZooCacheTest.java
index 39a2184568..3aced2e103 100644
--- 
a/core/src/test/java/org/apache/accumulo/core/fate/zookeeper/ZooCacheTest.java
+++ 
b/core/src/test/java/org/apache/accumulo/core/fate/zookeeper/ZooCacheTest.java
@@ -176,6 +176,8 @@ public class ZooCacheTest {
 
   @Test
   public void testGetChildren() throws Exception {
+    Stat existsStat = new Stat();
+    expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat);
     expect(zk.getChildren(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(CHILDREN);
     replay(zk);
 
@@ -190,34 +192,61 @@ public class ZooCacheTest {
 
   @Test
   public void testGetChildren_NoKids() throws Exception {
-    expect(zk.getChildren(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(null);
+    Stat existsStat = new Stat();
+    expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat);
+    expect(zk.getChildren(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(List.of());
     replay(zk);
 
-    assertNull(zc.getChildren(ZPATH));
+    assertEquals(List.of(), zc.getChildren(ZPATH));
     verify(zk);
 
-    assertNull(zc.getChildren(ZPATH)); // cache hit
+    assertEquals(List.of(), zc.getChildren(ZPATH)); // cache hit
+  }
+
+  @Test
+  public void testGetChildren_RaceCondition() throws Exception {
+    // simulate the node being deleted between calling zookeeper.exists and 
zookeeper.getChildren
+    Stat existsStat = new Stat();
+    expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat);
+    expect(zk.getChildren(eq(ZPATH), anyObject(Watcher.class)))
+        .andThrow(new KeeperException.NoNodeException(ZPATH));
+    expect(zk.exists(eq(ZPATH), anyObject(Watcher.class))).andReturn(null);
+    replay(zk);
+    assertNull(zc.getChildren(ZPATH));
+    verify(zk);
+    assertNull(zc.getChildren(ZPATH));
   }
 
   @Test
   public void testGetChildren_Retry() throws Exception {
+    Stat existsStat = new Stat();
+    expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat);
     expect(zk.getChildren(eq(ZPATH), anyObject(Watcher.class)))
         .andThrow(new KeeperException.BadVersionException(ZPATH));
+    expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat);
     expect(zk.getChildren(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(CHILDREN);
     replay(zk);
 
     assertEquals(CHILDREN, zc.getChildren(ZPATH));
     verify(zk);
+    assertEquals(CHILDREN, zc.getChildren(ZPATH));
   }
 
   @Test
-  public void testGetChildren_EatNoNode() throws Exception {
-    expect(zk.getChildren(eq(ZPATH), anyObject(Watcher.class)))
-        .andThrow(new KeeperException.NoNodeException(ZPATH));
+  public void testGetChildren_NoNode() throws Exception {
+    assertFalse(zc.childrenCached(ZPATH));
+    assertFalse(zc.dataCached(ZPATH));
+    expect(zk.exists(eq(ZPATH), anyObject(Watcher.class))).andReturn(null);
     replay(zk);
 
     assertNull(zc.getChildren(ZPATH));
     verify(zk);
+    assertNull(zc.getChildren(ZPATH));
+    // when its discovered a node does not exists in getChildren then its also 
known it does not
+    // exists for getData
+    assertNull(zc.get(ZPATH));
+    assertTrue(zc.childrenCached(ZPATH));
+    assertTrue(zc.dataCached(ZPATH));
   }
 
   private static class TestWatcher implements Watcher {
@@ -300,6 +329,82 @@ public class ZooCacheTest {
     testWatchDataNode_Clear(Watcher.Event.KeeperState.Expired);
   }
 
+  @Test
+  public void testGetDataThenChildren() throws Exception {
+    testGetBoth(true);
+  }
+
+  @Test
+  public void testGetChildrenThenDate() throws Exception {
+    testGetBoth(false);
+  }
+
+  private void testGetBoth(boolean getDataFirst) throws Exception {
+    assertFalse(zc.childrenCached(ZPATH));
+    assertFalse(zc.dataCached(ZPATH));
+
+    var uc1 = zc.getUpdateCount();
+
+    final long ephemeralOwner1 = 123456789L;
+    Stat existsStat1 = new Stat();
+    existsStat1.setEphemeralOwner(ephemeralOwner1);
+
+    final long ephemeralOwner2 = 987654321L;
+    Stat existsStat2 = new Stat();
+    existsStat2.setEphemeralOwner(ephemeralOwner2);
+
+    if (getDataFirst) {
+      expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat1);
+      expect(zk.getData(eq(ZPATH), anyObject(Watcher.class), 
eq(existsStat1))).andReturn(DATA);
+      expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat2);
+      expect(zk.getChildren(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(CHILDREN);
+    } else {
+      expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat2);
+      expect(zk.getChildren(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(CHILDREN);
+      expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat1);
+      expect(zk.getData(eq(ZPATH), anyObject(Watcher.class), 
eq(existsStat1))).andReturn(DATA);
+    }
+
+    replay(zk);
+
+    if (getDataFirst) {
+      var zcStat = new ZcStat();
+      var data = zc.get(ZPATH, zcStat);
+      assertEquals(ephemeralOwner1, zcStat.getEphemeralOwner());
+      assertArrayEquals(DATA, data);
+    } else {
+      var children = zc.getChildren(ZPATH);
+      assertEquals(CHILDREN, children);
+    }
+    var uc2 = zc.getUpdateCount();
+    assertTrue(uc1 < uc2);
+
+    if (getDataFirst) {
+      var children = zc.getChildren(ZPATH);
+      assertEquals(CHILDREN, children);
+    } else {
+      var zcStat = new ZcStat();
+      var data = zc.get(ZPATH, zcStat);
+      assertEquals(ephemeralOwner1, zcStat.getEphemeralOwner());
+      assertArrayEquals(DATA, data);
+    }
+    var uc3 = zc.getUpdateCount();
+    assertTrue(uc2 < uc3);
+
+    verify(zk);
+
+    var zcStat = new ZcStat();
+    var data = zc.get(ZPATH, zcStat);
+    // the stat is associated with the data so should aways see the one 
returned by the call to get
+    // data and not get children
+    assertEquals(ephemeralOwner1, zcStat.getEphemeralOwner());
+    assertArrayEquals(DATA, data);
+    var children = zc.getChildren(ZPATH);
+    assertEquals(CHILDREN, children);
+    // everything is cached so the get calls on the cache should not change 
the update count
+    assertEquals(uc3, zc.getUpdateCount());
+  }
+
   private void testWatchDataNode_Clear(Watcher.Event.KeeperState state) throws 
Exception {
     WatchedEvent event = new WatchedEvent(Watcher.Event.EventType.None, state, 
null);
     TestWatcher exw = new TestWatcher(event);
@@ -347,7 +452,13 @@ public class ZooCacheTest {
 
   private Watcher watchChildren(List<String> initialChildren) throws Exception 
{
     Capture<Watcher> cw = EasyMock.newCapture();
-    expect(zk.getChildren(eq(ZPATH), capture(cw))).andReturn(initialChildren);
+    if (initialChildren == null) {
+      expect(zk.exists(eq(ZPATH), capture(cw))).andReturn(null);
+    } else {
+      Stat existsStat = new Stat();
+      expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat);
+      expect(zk.getChildren(eq(ZPATH), 
capture(cw))).andReturn(initialChildren);
+    }
     replay(zk);
     zc.getChildren(ZPATH);
     assertTrue(zc.childrenCached(ZPATH));
diff --git 
a/test/src/main/java/org/apache/accumulo/test/zookeeper/ZooCacheIT.java 
b/test/src/main/java/org/apache/accumulo/test/zookeeper/ZooCacheIT.java
new file mode 100644
index 0000000000..645b7c70d2
--- /dev/null
+++ b/test/src/main/java/org/apache/accumulo/test/zookeeper/ZooCacheIT.java
@@ -0,0 +1,195 @@
+/*
+ * 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
+ *
+ *   https://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.accumulo.test.zookeeper;
+
+import static 
org.apache.accumulo.harness.AccumuloITBase.ZOOKEEPER_TESTING_SERVER;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertFalse;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertTrue;
+
+import java.io.File;
+import java.time.Duration;
+import java.util.Collections;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Set;
+
+import org.apache.accumulo.core.fate.zookeeper.ZooCache;
+import org.apache.accumulo.core.fate.zookeeper.ZooReaderWriter;
+import org.apache.accumulo.test.util.Wait;
+import org.apache.zookeeper.Watcher;
+import org.junit.jupiter.api.AfterEach;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Tag;
+import org.junit.jupiter.api.Test;
+import org.junit.jupiter.api.io.TempDir;
+
+@Tag(ZOOKEEPER_TESTING_SERVER)
+public class ZooCacheIT {
+
+  private ZooKeeperTestingServer szk = null;
+  private ZooReaderWriter zk = null;
+
+  @TempDir
+  private File tempDir;
+
+  @BeforeEach
+  public void setup() throws Exception {
+    szk = new ZooKeeperTestingServer(tempDir);
+    zk = szk.getZooReaderWriter();
+  }
+
+  @AfterEach
+  public void teardown() throws Exception {
+    szk.close();
+  }
+
+  @Test
+  public void testGetChildren() throws Exception {
+
+    Set<String> watchesRemoved = Collections.synchronizedSet(new HashSet<>());
+    Watcher watcher = event -> {
+      if (event.getType() == Watcher.Event.EventType.ChildWatchRemoved
+          || event.getType() == Watcher.Event.EventType.DataWatchRemoved) {
+        watchesRemoved.add(event.getPath());
+      }
+    };
+    ZooCache zooCache = new ZooCache(zk, watcher, Duration.ofSeconds(3));
+
+    zk.mkdirs("/test2");
+    zk.mkdirs("/test3/c1");
+    zk.mkdirs("/test3/c2");
+
+    // cache non-existence of /test1 and existence of /test2 and /test3
+    long uc1 = zooCache.getUpdateCount();
+    assertNull(zooCache.getChildren("/test1"));
+    long uc2 = zooCache.getUpdateCount();
+    assertTrue(uc1 < uc2);
+    assertEquals(List.of(), zooCache.getChildren("/test2"));
+    long uc3 = zooCache.getUpdateCount();
+    assertTrue(uc2 < uc3);
+    assertEquals(Set.of("c1", "c2"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    long uc4 = zooCache.getUpdateCount();
+    assertTrue(uc3 < uc4);
+
+    // The cache should be stable now and new accesses should not change the 
update count
+    assertNull(zooCache.getChildren("/test1"));
+    // once getChildren discovers that a node does not exists, then get data 
will also know this
+    assertNull(zooCache.get("/test1"));
+    assertEquals(List.of(), zooCache.getChildren("/test2"));
+    assertEquals(Set.of("c1", "c2"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    assertEquals(uc4, zooCache.getUpdateCount());
+
+    // Had cached non-existence of "/test1", should get a notification that it 
was created
+    zk.mkdirs("/test1");
+
+    Wait.waitFor(() -> {
+      var children = zooCache.getChildren("/test1");
+      return children != null && children.isEmpty();
+    });
+
+    long uc5 = zooCache.getUpdateCount();
+    assertTrue(uc4 < uc5);
+    assertEquals(List.of(), zooCache.getChildren("/test1"));
+    assertEquals(List.of(), zooCache.getChildren("/test2"));
+    assertEquals(Set.of("c1", "c2"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    assertEquals(uc5, zooCache.getUpdateCount());
+
+    // add a child to /test3, should get a notification of the change
+    zk.mkdirs("/test3/c3");
+    Wait.waitFor(() -> {
+      var children = zooCache.getChildren("/test3");
+      return children != null && children.size() == 3;
+    });
+    long uc6 = zooCache.getUpdateCount();
+    assertTrue(uc5 < uc6);
+    assertEquals(List.of(), zooCache.getChildren("/test1"));
+    assertEquals(List.of(), zooCache.getChildren("/test2"));
+    assertEquals(Set.of("c1", "c2", "c3"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    assertEquals(uc6, zooCache.getUpdateCount());
+
+    // remove a child from /test3
+    zk.delete("/test3/c2");
+    Wait.waitFor(() -> {
+      var children = zooCache.getChildren("/test3");
+      return children != null && children.size() == 2;
+    });
+    long uc7 = zooCache.getUpdateCount();
+    assertTrue(uc6 < uc7);
+    assertEquals(List.of(), zooCache.getChildren("/test1"));
+    assertEquals(List.of(), zooCache.getChildren("/test2"));
+    assertEquals(Set.of("c1", "c3"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    assertEquals(uc7, zooCache.getUpdateCount());
+
+    // remove /test2, should start caching that it does not exist
+    zk.delete("/test2");
+    Wait.waitFor(() -> zooCache.getChildren("/test2") == null);
+    long uc8 = zooCache.getUpdateCount();
+    assertTrue(uc7 < uc8);
+    assertEquals(List.of(), zooCache.getChildren("/test1"));
+    assertNull(zooCache.getChildren("/test2"));
+    assertEquals(Set.of("c1", "c3"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    assertEquals(uc8, zooCache.getUpdateCount());
+
+    // add /test2 back, should update
+    zk.mkdirs("/test2");
+    Wait.waitFor(() -> zooCache.getChildren("/test2") != null);
+    long uc9 = zooCache.getUpdateCount();
+    assertTrue(uc8 < uc9);
+    assertEquals(List.of(), zooCache.getChildren("/test1"));
+    assertEquals(List.of(), zooCache.getChildren("/test2"));
+    assertEquals(Set.of("c1", "c3"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    assertEquals(uc9, zooCache.getUpdateCount());
+
+    // make multiple changes. the cache should see all of these
+    zk.delete("/test1");
+    zk.mkdirs("/test2/ca");
+    zk.delete("/test3/c1");
+    zk.mkdirs("/test3/c4");
+    zk.delete("/test3/c4");
+    zk.mkdirs("/test3/c5");
+
+    Wait.waitFor(() -> {
+      var children1 = zooCache.getChildren("/test1");
+      var children2 = zooCache.getChildren("/test2");
+      var children3 = zooCache.getChildren("/test3");
+      return children1 == null && children2 != null && children2.size() == 1 
&& children3 != null
+          && Set.copyOf(children3).equals(Set.of("c3", "c5"));
+    });
+    long uc10 = zooCache.getUpdateCount();
+    assertTrue(uc9 < uc10);
+    assertNull(zooCache.getChildren("/test1"));
+    assertEquals(List.of("ca"), zooCache.getChildren("/test2"));
+    assertEquals(Set.of("c3", "c5"), 
Set.copyOf(zooCache.getChildren("/test3")));
+    assertEquals(uc10, zooCache.getUpdateCount());
+
+    // wait for the cache to evict and clear watches
+    Wait.waitFor(() -> {
+      // the cache will not run its eviction handler unless accessed, so 
access something that is
+      // not expected to be evicted
+      zooCache.getChildren("/test4");
+      return watchesRemoved.equals(Set.of("/test1", "/test2", "/test3"));
+    });
+
+    assertFalse(zooCache.childrenCached("/test1"));
+    assertFalse(zooCache.childrenCached("/test2"));
+    assertFalse(zooCache.childrenCached("/test3"));
+  }
+}

Reply via email to