ctubbsii commented on code in PR #5256:
URL: https://github.com/apache/accumulo/pull/5256#discussion_r1928667390


##########
core/src/main/java/org/apache/accumulo/core/lock/ServiceLock.java:
##########
@@ -672,7 +672,7 @@ public static Optional<ServiceLockData> 
getLockData(ZooSession zk, ServiceLockPa
   }
 
   public static Optional<ServiceLockData> getLockData(
-      org.apache.accumulo.core.fate.zookeeper.ZooCache zc, ServiceLockPath 
path, ZcStat stat) {
+      org.apache.accumulo.core.zookeeper.ZooCache zc, ServiceLockPath path, 
ZcStat stat) {

Review Comment:
   ```suggestion
         ZooCache zc, ServiceLockPath path, ZcStat stat) {
   ```



##########
core/src/test/java/org/apache/accumulo/core/zookeeper/ZooCacheTest.java:
##########
@@ -16,48 +16,111 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.accumulo.core.fate.zookeeper;
+package org.apache.accumulo.core.zookeeper;
 
 import static org.easymock.EasyMock.anyObject;
-import static org.easymock.EasyMock.capture;
 import static org.easymock.EasyMock.createStrictMock;
 import static org.easymock.EasyMock.eq;
 import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.isA;
 import static org.easymock.EasyMock.replay;
+import static org.easymock.EasyMock.reset;
 import static org.easymock.EasyMock.verify;
 import static org.junit.jupiter.api.Assertions.assertArrayEquals;
 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.assertSame;
+import static org.junit.jupiter.api.Assertions.assertThrows;
 import static org.junit.jupiter.api.Assertions.assertTrue;
 
+import java.util.Collections;
 import java.util.List;
+import java.util.Set;
+import java.util.TreeSet;
+import java.util.UUID;
 
-import org.apache.accumulo.core.fate.zookeeper.ZooCache.ZcStat;
-import org.apache.accumulo.core.fate.zookeeper.ZooCache.ZooCacheWatcher;
-import org.apache.accumulo.core.zookeeper.ZooSession;
+import org.apache.accumulo.core.Constants;
+import org.apache.accumulo.core.zookeeper.ZooCache.ZooCacheWatcher;
 import org.apache.zookeeper.KeeperException;
 import org.apache.zookeeper.WatchedEvent;
 import org.apache.zookeeper.Watcher;
 import org.apache.zookeeper.data.Stat;
-import org.easymock.Capture;
-import org.easymock.EasyMock;
 import org.junit.jupiter.api.BeforeEach;
 import org.junit.jupiter.api.Test;
 
 public class ZooCacheTest {
-  private static final String ZPATH = "/some/path/in/zk";
+
+  /**
+   * Test class that extends ZooCache to suppress the creation of the 
persistent recursive watchers
+   * that are created in the constructor and to provide access to the watcher.
+   */
+  private static class TestZooCache extends ZooCache {
+
+    public TestZooCache(ZooSession zk, Set<String> pathsToWatch) {
+      super(zk, pathsToWatch);
+    }
+
+    @Override
+    protected void setupWatchers(Set<String> pathsToWatch) {
+      TreeSet<String> wPaths = new TreeSet<>();
+      for (String path : pathsToWatch) {
+        wPaths.add(path);
+      }
+      watchedPaths = Collections.unmodifiableNavigableSet(wPaths);
+    }
+
+    public void executeWatcher(WatchedEvent event) {
+      // simulate ZooKeeper calling our Watcher
+      watcher.process(event);

Review Comment:
   We have a suite of tests that creates a real ZooKeeper for integration 
testing. It's at 
test/src/main/java/org/apache/accumulo/test/zookeeper/ZooKeeperTestingServer.java
 ; You could use that same to trigger actual watch events as an integration 
test.



##########
minicluster/src/main/java/org/apache/accumulo/miniclusterImpl/MiniAccumuloClusterImpl.java:
##########
@@ -1003,31 +1002,23 @@ public synchronized void stop() throws IOException, 
InterruptedException {
     } catch (RuntimeException e) {
       log.error("Error zapping zookeeper locks", e);
     }
-    control.stop(ServerType.ZOOKEEPER, null);
 
     // Clear the location of the servers in ZooCache.
-    // When ZooKeeper was stopped in the previous method call,
-    // the local ZooKeeper watcher did not fire. If MAC is
-    // restarted, then ZooKeeper will start on the same port with
-    // the same data, but no Watchers will fire.
-    boolean startCalled = true;
+    // If MAC is restarted, then ZooKeeper will start
+    // on the same port with the same data, but no Watchers will fire.

Review Comment:
   I don't think that's true. The watchers should see the disconnected event 
when  MAC is stopped, and either a syncconnected or an expired session, 
depending on how long it is offline when it restarts and whether that time 
exceeds the connect timeout.
   
   The real reason for clearing the locations is that the ephemeral nodes for 
the locks corresponding to the stopped Accumulo processes will still exist, 
because the ZK session still exists, but since these services were restarted, 
they need to create new locks.
   
   So, it's not about the watchers not firing... it's just that the locks are 
no longer up-to-date with the currently running Accumulo services.



##########
core/src/test/java/org/apache/accumulo/core/zookeeper/ZooCacheTest.java:
##########
@@ -78,8 +141,8 @@ private void testGet(boolean fillStat) throws Exception {
     final long ephemeralOwner = 123456789L;
     Stat existsStat = new Stat();
     existsStat.setEphemeralOwner(ephemeralOwner);
-    expect(zk.exists(eq(ZPATH), 
anyObject(Watcher.class))).andReturn(existsStat);
-    expect(zk.getData(eq(ZPATH), anyObject(Watcher.class), 
eq(existsStat))).andReturn(DATA);
+    expect(zk.exists(eq(ZPATH), eq(null))).andReturn(existsStat);

Review Comment:
   If you're just using literal values, you don't need to use matchers. You 
just have to be consistent, either all of the params use a matcher, or none do. 
In this case, we're expecting very specific values, so matchers aren't needed.
   
   ```suggestion
       expect(zk.exists(ZPATH, null)).andReturn(existsStat);
   ```
   
   It's fine to use matchers even when they're not needed, but in some places 
(not here, because this is simple enough), it can make the test code look way 
more complicated than it actually is. So, I tend to avoid them when not 
strictly needed.



##########
pom.xml:
##########
@@ -688,6 +688,7 @@
             <effort>Max</effort>
             <failOnError>true</failOnError>
             <includeTests>true</includeTests>
+            <maxHeap>1024</maxHeap>

Review Comment:
   Does this help solve a build problem, or were you just trying different 
things and left this in?



##########
core/src/main/java/org/apache/accumulo/core/zookeeper/ZooCache.java:
##########
@@ -59,77 +62,72 @@ public interface ZooCacheWatcher extends 
Consumer<WatchedEvent> {}
 
   private static final Logger log = LoggerFactory.getLogger(ZooCache.class);
 
-  private final ZCacheWatcher watcher = new ZCacheWatcher();
-  private final Optional<ZooCacheWatcher> externalWatcher;
+  protected volatile NavigableSet<String> watchedPaths =
+      Collections.unmodifiableNavigableSet(new TreeSet<>());

Review Comment:
   The lifecycle of this field is confusing, because it is initialized to an 
empty set here, then in the constructor, setupWatchers is called, which sets 
this again to the set passed into the constructor. Then, when ZK connection 
changes are detected, setupWatchers is called again, using the contents of this 
set as input, which again clobbers this field with the same set.
   
   Instead, just make this a final, set it once in the constructor (as a copy 
of the input set), and use it in setupWatchers directly, without modifying it, 
and without the need to pass it as a parameter to setupWatchers (because it's 
already an instance member and setupWatchers is a non-static instance method).



##########
server/base/src/main/java/org/apache/accumulo/server/tables/TableManager.java:
##########
@@ -286,11 +285,13 @@ public void accept(WatchedEvent event) {
         case NodeCreated:
         case NodeDataChanged:
           // state transition
-          TableState tState = updateTableStateCache(tableId);
-          log.debug("State transition to {} @ {}", tState, event);
-          synchronized (observers) {
-            for (TableObserver to : observers) {
-              to.stateChanged(tableId, tState);
+          if (tableId != null) {
+            TableState tState = updateTableStateCache(tableId);
+            log.debug("State transition to {} @ {}", tState, event);
+            synchronized (observers) {
+              for (TableObserver to : observers) {
+                to.stateChanged(tableId, tState);
+              }
             }
           }

Review Comment:
   Is this always going to be a table state change? Or could it be a different 
kind of watched event on that table path? This code seems to assume that it's 
always a state change, but I'm not sure if that's true anymore.



##########
core/src/main/java/org/apache/accumulo/core/zookeeper/ZooCache.java:
##########
@@ -143,65 +141,106 @@ public void process(WatchedEvent event) {
               clear();
               break;
             default:
-              log.warn("{} Unhandled {}", cacheId, event);
+              log.warn("{} Unhandled state {}", cacheId, event);
               break;
           }
           break;
         default:
-          log.warn("{} Unhandled {}", cacheId, event);
+          log.warn("{} Unhandled event type {}", cacheId, event);
           break;
       }
 
-      externalWatcher.ifPresent(w -> w.accept(event));
+      externalWatchers.forEach(ew -> ew.accept(event));
     }
   }
 
   /**
-   * Creates a new cache without an external watcher.
+   * Creates a ZooCache instance that uses the supplied ZooSession for 
communicating with the
+   * instance's ZooKeeper servers. The ZooCache will create persistent 
watchers at the given
+   * pathsToWatch, if any, to be updated when changes are made in ZooKeeper 
for nodes at or below in
+   * the tree. If ZooCacheWatcher's are added via {@code addZooCacheWatcher}, 
then they will be
+   * notified when this object is notified of changes via the 
PersistentWatcher callback.
    *
-   * @param zk the ZooKeeper instance
-   * @throws NullPointerException if zk is {@code null}
+   * @param zk ZooSession for this instance
+   * @param pathsToWatch Paths in ZooKeeper to watch
    */
-  public ZooCache(ZooSession zk) {
-    this(zk, Optional.empty(), Duration.ofMinutes(3));
+  public ZooCache(ZooSession zk, Set<String> pathsToWatch) {
+    this(zk, pathsToWatch, Ticker.systemTicker());
   }
 
-  /**
-   * Creates a new cache. The given watcher is called whenever a watched node 
changes.
-   *
-   * @param zk the ZooKeeper instance
-   * @param watcher watcher object
-   * @throws NullPointerException if zk or watcher is {@code null}
-   */
-  public ZooCache(ZooSession zk, ZooCacheWatcher watcher) {
-    this(zk, Optional.of(watcher), Duration.ofMinutes(3));
+  // visible for tests that use a Ticker
+  public ZooCache(ZooSession zk, Set<String> pathsToWatch, Ticker ticker) {
+    this.zk = requireNonNull(zk);
+    this.zkClientTracker.set(this.getZKClientObjectVersion());
+    this.cache = 
Caches.getInstance().createNewBuilder(Caches.CacheName.ZOO_CACHE, false)
+        
.ticker(requireNonNull(ticker)).expireAfterAccess(CACHE_DURATION).build();
+    // 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.
+    this.nodeCache = cache.asMap();
+    setupWatchers(pathsToWatch);
+    log.trace("{} created new cache", cacheId, new Exception());
   }
 
-  public ZooCache(ZooSession zk, Optional<ZooCacheWatcher> watcher, Duration 
timeout) {
-    this.zk = requireNonNull(zk);
-    this.externalWatcher = watcher;
-    RemovalListener<String,ZcNode> removalListerner = (path, zcNode, reason) 
-> {
-      try {
-        log.trace("{} removing watches for {} because {} accesses {}", 
cacheId, path, reason,
-            zcNode == null ? -1 : zcNode.getAccessCount());
-        zk.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);
+  public void addZooCacheWatcher(ZooCacheWatcher watcher) {
+    externalWatchers.add(requireNonNull(watcher));
+  }
+
+  // visible for tests
+  long getZKClientObjectVersion() {
+    return zk.getConnectionCounter();
+  }
+
+  private boolean handleZKConnectionChange() {
+    final long currentCount = getZKClientObjectVersion();
+    final long oldCount = zkClientTracker.get();
+    if (oldCount != currentCount) {
+      if (zkClientTracker.compareAndSet(oldCount, currentCount)) {
+        setupWatchers(watchedPaths);
       }
-    };
-    // 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());
+      return true;
+    }
+    return false;
   }
 
-  private abstract static class ZooRunnable<T> {
+  // Called on construction and when ZooKeeper connection changes
+  synchronized void setupWatchers(Set<String> pathsToWatch) {
+
+    for (String left : pathsToWatch) {
+      for (String right : pathsToWatch) {
+        if (!left.equals(right) && left.contains(right)) {
+          throw new IllegalArgumentException(
+              "Overlapping paths found in paths to watch. left: " + left + ", 
right: " + right);
+        }
+      }
+    }
+
+    try {
+      zk.addPersistentRecursiveWatchers(pathsToWatch, watcher);
+      clear();
+      watchedPaths = Collections.unmodifiableNavigableSet(new 
TreeSet<>(pathsToWatch));
+    } catch (KeeperException | InterruptedException e) {
+      throw new RuntimeException("Error setting up persistent recursive 
watcher", e);
+    }
+
+  }
+
+  private boolean isWatchedPath(String path) {
+    // Check that the path is equal to, or a descendant of, a watched path
+    var floor = watchedPaths.floor(path);
+    return floor != null
+        && (floor.equals("/") || floor.equals(path) || path.startsWith(floor + 
"/"));
+  }
+
+  // Use this instead of Preconditions.checkState(isWatchedPath, String)
+  // so that we are not creating String unnecessarily.

Review Comment:
   This comment feels like an apology to Guava for not using one of its 
methods, but Preconditions isn't the presumed default way to write this simple 
if statement, so we owe it no explanation.
   
   That said, there is a version of Preconditions.checkState that takes a 
format string and format parameters, to construct the string only if the 
exception needs to be thrown. So, this comment isn't really necessary. We can 
either use that version of Preconditions.checkState (or maybe a newer version 
that takes a `Supplier<String>` if that exists), or just keep this method like 
it is, which generally speaks for itself without the comment.



##########
core/src/main/java/org/apache/accumulo/core/zookeeper/ZooCache.java:
##########
@@ -59,77 +62,72 @@ public interface ZooCacheWatcher extends 
Consumer<WatchedEvent> {}
 
   private static final Logger log = LoggerFactory.getLogger(ZooCache.class);
 
-  private final ZCacheWatcher watcher = new ZCacheWatcher();
-  private final Optional<ZooCacheWatcher> externalWatcher;
+  protected volatile NavigableSet<String> watchedPaths =
+      Collections.unmodifiableNavigableSet(new TreeSet<>());
+  // visible for tests
+  protected final ZCacheWatcher watcher = new ZCacheWatcher();
+  private final List<ZooCacheWatcher> externalWatchers =
+      Collections.synchronizedList(new ArrayList<>());
 
   private static final AtomicLong nextCacheId = new AtomicLong(0);
   private final String cacheId = "ZC" + nextCacheId.incrementAndGet();
 
-  // 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.
+  public static final Duration CACHE_DURATION = Duration.ofMinutes(30);
+
+  private final Cache<String,ZcNode> cache;
+
   private final ConcurrentMap<String,ZcNode> nodeCache;
 
   private final ZooSession zk;
 
   private volatile boolean closed = false;
 
-  public static class ZcStat {
-    private long ephemeralOwner;
-    private long mzxid;
+  private final AtomicLong updateCount = new AtomicLong();
 
-    public ZcStat() {}
+  private final AtomicLong zkClientTracker = new AtomicLong();
 
-    private ZcStat(Stat stat) {
-      this.ephemeralOwner = stat.getEphemeralOwner();
-      this.mzxid = stat.getMzxid();
-    }
-
-    public long getEphemeralOwner() {
-      return ephemeralOwner;
-    }
-
-    private void set(ZcStat cachedStat) {
-      this.ephemeralOwner = cachedStat.ephemeralOwner;
-      this.mzxid = cachedStat.mzxid;
-    }
-
-    @VisibleForTesting
-    public void setEphemeralOwner(long ephemeralOwner) {
-      this.ephemeralOwner = ephemeralOwner;
-    }
-
-    public long getMzxid() {
-      return mzxid;
-    }
-  }
-
-  private final AtomicLong updateCount = new AtomicLong(0);
-
-  private class ZCacheWatcher implements Watcher {
+  class ZCacheWatcher implements Watcher {
     @Override
     public void process(WatchedEvent event) {
       if (log.isTraceEnabled()) {
         log.trace("{}: {}", cacheId, event);
       }
 
       switch (event.getType()) {
-        case NodeDataChanged:
         case NodeChildrenChanged:
-        case NodeCreated:
-        case NodeDeleted:
+          // According to documentation we should not receive this event.
+          // According to https://issues.apache.org/jira/browse/ZOOKEEPER-4475 
we
+          // may receive this event (Fixed in 3.9.0)
+          break;
         case ChildWatchRemoved:
         case DataWatchRemoved:
-          remove(event.getPath());
+          // We don't need to do anything with the cache on these events.
+          break;
+        case NodeDataChanged:
+          log.trace("{} node data changed; clearing {}", cacheId, 
event.getPath());
+          clear(path -> path.equals(event.getPath()));
+          break;
+        case NodeCreated:
+        case NodeDeleted:
+          // With the Watcher being set at a higher level we need to remove
+          // the parent of the affected node and all of its children from the 
cache
+          // so that the parent and children node can be re-cached. If we only 
remove the
+          // affected node, then the cached children in the parent could be 
incorrect.
+          int lastSlash = event.getPath().lastIndexOf('/');
+          String parent = lastSlash == 0 ? "/" : event.getPath().substring(0, 
lastSlash);
+          log.trace("{} node created or deleted {}; clearing {}", cacheId, 
event.getPath(), parent);
+          clear((path) -> path.startsWith(parent));
+          break;
+        case PersistentWatchRemoved:
+          log.trace(
+              "{} persistent watch removed {} which is only done in 
ZooSession.addPersistentRecursiveWatchers; ignoring;",
+              cacheId, event.getPath());

Review Comment:
   This is no longer expected from ZooSession



##########
core/src/test/java/org/apache/accumulo/core/zookeeper/ZooCacheTest.java:
##########
@@ -287,29 +350,31 @@ private void testWatchDataNode(byte[] initialData, 
Watcher.Event.EventType event
     WatchedEvent event =
         new WatchedEvent(eventType, Watcher.Event.KeeperState.SyncConnected, 
ZPATH);
     TestWatcher exw = new TestWatcher(event);
-    zc = new ZooCache(zk, exw);
-
-    Watcher w = watchData(initialData);
-    w.process(event);
+    zc = new TestZooCache(zk, Set.of(root));
+    zc.addZooCacheWatcher(exw);
+    // The ZooCache constructor sets an expectation
+    // of addSessionObserver being called later during
+    // the tests. Remove this expectation by calling
+    // reset

Review Comment:
   The session observer stuff was reverted from the PR. So this comment, and 
perhaps the reset also, are no longer needed.



##########
server/base/src/main/java/org/apache/accumulo/server/security/handler/KerberosAuthenticator.java:
##########
@@ -71,35 +68,30 @@ public boolean validSecurityHandlers() {
   }
 
   private void createUserNodeInZk(String principal) throws KeeperException, 
InterruptedException {
-    synchronized (zooCache) {
-      zooCache.clear();
-      ZooReaderWriter zoo = context.getZooSession().asReaderWriter();
-      zoo.putPrivatePersistentData(zkUserPath + "/" + principal, new byte[0],
-          NodeExistsPolicy.FAIL);
-    }
+    context.getZooCache().clear(zkUserPath + "/" + principal);
+    ZooReaderWriter zoo = context.getZooSession().asReaderWriter();
+    zoo.putPrivatePersistentData(zkUserPath + "/" + principal, new byte[0], 
NodeExistsPolicy.FAIL);
   }
 
   @Override
   public void initializeSecurity(String principal, byte[] token) {
     try {
       // remove old settings from zookeeper first, if any
       ZooReaderWriter zoo = context.getZooSession().asReaderWriter();
-      synchronized (zooCache) {
-        zooCache.clear();
-        if (zoo.exists(zkUserPath)) {
-          zoo.recursiveDelete(zkUserPath, NodeMissingPolicy.SKIP);
-          log.info("Removed {}/ from zookeeper", zkUserPath);
-        }
-
-        // prep parent node of users with root username
-        // ACCUMULO-4140 The root user needs to be stored un-base64 encoded in 
the znode's value
-        byte[] principalData = principal.getBytes(UTF_8);
-        zoo.putPersistentData(zkUserPath, principalData, 
NodeExistsPolicy.FAIL);
-
-        // Create the root user in ZK using base64 encoded name (since the 
name is included in the
-        // znode)
-        createUserNodeInZk(Base64.getEncoder().encodeToString(principalData));
+      context.getZooCache().clear((path) -> path.startsWith(zkUserPath));

Review Comment:
   It looks like these synchronized blocks were dependent on previous ZooCache 
implementation that synchronized the reads from ZooCache. That is no longer the 
case, so these should be revisited as a follow-on. This code is old and I'm not 
sure what it was trying to do clearing the ZooCache. Presumably it was so any 
permissions/auth changes were immediately consistent within the same process. 
However, if that's the case, then clear needs to take place after the writes to 
ZooKeeper, not before, since ZooCache read methods are no longer synchronized.
   
   This is probably a bug in all branches, and should be fixed in 2.1 and later.



##########
minicluster/src/main/java/org/apache/accumulo/miniclusterImpl/MiniAccumuloClusterImpl.java:
##########
@@ -994,7 +993,7 @@ public synchronized void stop() throws IOException, 
InterruptedException {
     control.stop(ServerType.SCAN_SERVER, null);
 
     // The method calls above kill the server
-    // Clean up the locks in ZooKeeper fo that if the cluster
+    // Clean up the locks in ZooKeeper for that if the cluster

Review Comment:
   I'm not sure this sentence makes any more sense after this fix. "for that if 
the cluster is restarted" is a very confusing statement, and I'm not sure what 
it's trying to say.



##########
server/base/src/test/java/org/apache/accumulo/server/security/handler/ZKAuthenticatorTest.java:
##########
@@ -144,14 +145,16 @@ public void testUserAuthentication() throws Exception {
     var instanceId = InstanceId.of("example");
     ZooSession zk = createMock(ZooSession.class);
     ServerContext context = MockServerContext.getWithMockZK(zk);
+    ZooCache zc = createMock(ZooCache.class);
     expect(context.zkUserPath()).andReturn(ZooUtil.getRoot(instanceId) + 
Constants.ZUSERS)
         .anyTimes();
     expect(zk.getChildren(anyObject(), 
anyObject())).andReturn(Arrays.asList(principal)).anyTimes();
     expect(zk.exists(matches(ZooUtil.getRoot(instanceId) + Constants.ZUSERS + 
"/" + principal),
         anyObject(Watcher.class))).andReturn(new Stat()).anyTimes();
-    expect(zk.getData(matches(ZooUtil.getRoot(instanceId) + Constants.ZUSERS + 
"/" + principal),
-        anyObject(), anyObject())).andReturn(newHash).once();
-    replay(context, zk);
+    expect(context.getZooCache()).andReturn(zc).anyTimes();
+    expect(zc.get(matches(ZooUtil.getRoot(instanceId) + Constants.ZUSERS + "/" 
+ principal)))

Review Comment:
   Is 'matches()' the same as 'eq()'? I don't think I've used that before. In 
any case, this doesn't seem to need a matcher since we're matching on an exact 
value.
   
   ```suggestion
       expect(zc.get(ZooUtil.getRoot(instanceId) + Constants.ZUSERS + "/" + 
principal))
   ```



##########
core/src/main/java/org/apache/accumulo/core/zookeeper/ZooCache.java:
##########
@@ -143,65 +141,106 @@ public void process(WatchedEvent event) {
               clear();
               break;
             default:
-              log.warn("{} Unhandled {}", cacheId, event);
+              log.warn("{} Unhandled state {}", cacheId, event);
               break;
           }
           break;
         default:
-          log.warn("{} Unhandled {}", cacheId, event);
+          log.warn("{} Unhandled event type {}", cacheId, event);
           break;
       }
 
-      externalWatcher.ifPresent(w -> w.accept(event));
+      externalWatchers.forEach(ew -> ew.accept(event));
     }
   }
 
   /**
-   * Creates a new cache without an external watcher.
+   * Creates a ZooCache instance that uses the supplied ZooSession for 
communicating with the
+   * instance's ZooKeeper servers. The ZooCache will create persistent 
watchers at the given
+   * pathsToWatch, if any, to be updated when changes are made in ZooKeeper 
for nodes at or below in
+   * the tree. If ZooCacheWatcher's are added via {@code addZooCacheWatcher}, 
then they will be
+   * notified when this object is notified of changes via the 
PersistentWatcher callback.
    *
-   * @param zk the ZooKeeper instance
-   * @throws NullPointerException if zk is {@code null}
+   * @param zk ZooSession for this instance
+   * @param pathsToWatch Paths in ZooKeeper to watch
    */
-  public ZooCache(ZooSession zk) {
-    this(zk, Optional.empty(), Duration.ofMinutes(3));
+  public ZooCache(ZooSession zk, Set<String> pathsToWatch) {
+    this(zk, pathsToWatch, Ticker.systemTicker());
   }
 
-  /**
-   * Creates a new cache. The given watcher is called whenever a watched node 
changes.
-   *
-   * @param zk the ZooKeeper instance
-   * @param watcher watcher object
-   * @throws NullPointerException if zk or watcher is {@code null}
-   */
-  public ZooCache(ZooSession zk, ZooCacheWatcher watcher) {
-    this(zk, Optional.of(watcher), Duration.ofMinutes(3));
+  // visible for tests that use a Ticker
+  public ZooCache(ZooSession zk, Set<String> pathsToWatch, Ticker ticker) {
+    this.zk = requireNonNull(zk);
+    this.zkClientTracker.set(this.getZKClientObjectVersion());
+    this.cache = 
Caches.getInstance().createNewBuilder(Caches.CacheName.ZOO_CACHE, false)
+        
.ticker(requireNonNull(ticker)).expireAfterAccess(CACHE_DURATION).build();
+    // The concurrent map returned by Caffiene will only allow one thread to 
run at a time for a

Review Comment:
   ```suggestion
       // The concurrent map returned by Caffeine will only allow one thread to 
run at a time for a
   ```



##########
core/src/main/java/org/apache/accumulo/core/zookeeper/ZooCache.java:
##########
@@ -143,65 +141,106 @@ public void process(WatchedEvent event) {
               clear();
               break;
             default:
-              log.warn("{} Unhandled {}", cacheId, event);
+              log.warn("{} Unhandled state {}", cacheId, event);
               break;
           }
           break;
         default:
-          log.warn("{} Unhandled {}", cacheId, event);
+          log.warn("{} Unhandled event type {}", cacheId, event);
           break;
       }
 
-      externalWatcher.ifPresent(w -> w.accept(event));
+      externalWatchers.forEach(ew -> ew.accept(event));
     }
   }
 
   /**
-   * Creates a new cache without an external watcher.
+   * Creates a ZooCache instance that uses the supplied ZooSession for 
communicating with the
+   * instance's ZooKeeper servers. The ZooCache will create persistent 
watchers at the given
+   * pathsToWatch, if any, to be updated when changes are made in ZooKeeper 
for nodes at or below in
+   * the tree. If ZooCacheWatcher's are added via {@code addZooCacheWatcher}, 
then they will be
+   * notified when this object is notified of changes via the 
PersistentWatcher callback.
    *
-   * @param zk the ZooKeeper instance
-   * @throws NullPointerException if zk is {@code null}
+   * @param zk ZooSession for this instance
+   * @param pathsToWatch Paths in ZooKeeper to watch
    */
-  public ZooCache(ZooSession zk) {
-    this(zk, Optional.empty(), Duration.ofMinutes(3));
+  public ZooCache(ZooSession zk, Set<String> pathsToWatch) {
+    this(zk, pathsToWatch, Ticker.systemTicker());
   }
 
-  /**
-   * Creates a new cache. The given watcher is called whenever a watched node 
changes.
-   *
-   * @param zk the ZooKeeper instance
-   * @param watcher watcher object
-   * @throws NullPointerException if zk or watcher is {@code null}
-   */
-  public ZooCache(ZooSession zk, ZooCacheWatcher watcher) {
-    this(zk, Optional.of(watcher), Duration.ofMinutes(3));
+  // visible for tests that use a Ticker
+  public ZooCache(ZooSession zk, Set<String> pathsToWatch, Ticker ticker) {
+    this.zk = requireNonNull(zk);
+    this.zkClientTracker.set(this.getZKClientObjectVersion());
+    this.cache = 
Caches.getInstance().createNewBuilder(Caches.CacheName.ZOO_CACHE, false)
+        
.ticker(requireNonNull(ticker)).expireAfterAccess(CACHE_DURATION).build();
+    // 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.
+    this.nodeCache = cache.asMap();
+    setupWatchers(pathsToWatch);
+    log.trace("{} created new cache", cacheId, new Exception());
   }
 
-  public ZooCache(ZooSession zk, Optional<ZooCacheWatcher> watcher, Duration 
timeout) {
-    this.zk = requireNonNull(zk);
-    this.externalWatcher = watcher;
-    RemovalListener<String,ZcNode> removalListerner = (path, zcNode, reason) 
-> {
-      try {
-        log.trace("{} removing watches for {} because {} accesses {}", 
cacheId, path, reason,
-            zcNode == null ? -1 : zcNode.getAccessCount());
-        zk.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);
+  public void addZooCacheWatcher(ZooCacheWatcher watcher) {
+    externalWatchers.add(requireNonNull(watcher));
+  }
+
+  // visible for tests
+  long getZKClientObjectVersion() {
+    return zk.getConnectionCounter();
+  }
+
+  private boolean handleZKConnectionChange() {

Review Comment:
   ```suggestion
     /**
      * @return true if ZK has changed; false otherwise
      */
     private boolean handleZKConnectionChange() {
   ```



-- 
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.

To unsubscribe, e-mail: notifications-unsubscr...@accumulo.apache.org

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

Reply via email to