Merge branch 'stable-3.7' into stable-3.8

* stable-3.7:
  Split integrations tests into separate Bazel targets.
  Extract base test class for regular and async acceptance tests
  Allow additional refs fallback to apply-objects
  Stop creating redundant replication tasks from stream-events
  Add event creation time to the apply object payload
  Don't read git submodule commits during replication
  Expose implicit dependency with the multi-site plugin
  Ignore remote ref-updated stream events for replication
  Increase test time to 900 seconds
  Revert "Add documentation regarding Bearer Token Authentication set up."
  Dont copy pull-replication.jar into lib folder in Dockerfile
  Adapt to the renamed events-broker.jar build artifact
  Don't shade org.eclipse.jgit.transport package
  Fix eclipse project generation
  Fix entrypoint.sh in example-setup with broker
  Add documentation regarding Bearer Token Authentication set up.
  entrypoint.sh: Fix setting JAVA_OPTS variable
  Make the code build on Java 8
  Run apply-object before the ref-updated stream event
  Add extra logging when replication tasks are merged

Change-Id: Idcdc19e05ab11bb0c963da3d9617b3c3fa979a35
diff --git a/BUILD b/BUILD
index f6a6e48..7646282 100644
--- a/BUILD
+++ b/BUILD
@@ -25,7 +25,6 @@
     name = "pull_replication_tests",
     srcs = glob([
         "src/test/java/**/*Test.java",
-        "src/test/java/**/*IT.java",
     ]),
     tags = ["pull-replication"],
     visibility = ["//visibility:public"],
@@ -37,6 +36,18 @@
     ],
 )
 
+[junit_tests(
+    name = f[:f.index(".")].replace("/", "_"),
+    srcs = [f],
+    tags = ["pull-replication"],
+    visibility = ["//visibility:public"],
+    deps = PLUGIN_TEST_DEPS + PLUGIN_DEPS + [
+        ":pull-replication__plugin",
+        ":pull_replication_util",
+        "//plugins/replication",
+    ],
+) for f in glob(["src/test/java/**/*IT.java"])]
+
 java_library(
     name = "pull_replication_util",
     testonly = True,
@@ -49,6 +60,16 @@
     ),
     deps = PLUGIN_TEST_DEPS + PLUGIN_DEPS + [
         ":pull-replication__plugin",
-        "//plugins/replication:replication",
+        "//plugins/replication",
+    ],
+)
+
+java_library(
+    name = "pull-replication__plugin_test_deps",
+    testonly = 1,
+    visibility = ["//visibility:public"],
+    exports = PLUGIN_DEPS + PLUGIN_TEST_DEPS + [
+        ":pull-replication__plugin",
+        "@events-broker//jar",
     ],
 )
diff --git a/example-setup/broker/Dockerfile b/example-setup/broker/Dockerfile
index 0f96cb9..67eecd9 100644
--- a/example-setup/broker/Dockerfile
+++ b/example-setup/broker/Dockerfile
@@ -8,7 +8,10 @@
 
 COPY --chown=gerrit:gerrit pull-replication.jar /var/gerrit/plugins/pull-replication.jar
 
-COPY --chown=gerrit:gerrit events-kafka.jar /var/gerrit/plugins/events-kafka.jar
+# The message-broker notification needs to be the last in the notification chain
+# hence rename it with a 'z-' prefix because the Gerrit plugin loader starts the
+# plugins in filename alphabetical order.
+COPY --chown=gerrit:gerrit events-kafka.jar /var/gerrit/plugins/z-events-kafka.jar
 COPY --chown=gerrit:gerrit events-broker.jar /var/gerrit/lib/events-broker.jar
 
 COPY --chown=gerrit:gerrit entrypoint.sh /tmp/
diff --git a/example-setup/broker/configs/replication.config.template b/example-setup/broker/configs/replication.config.template
index d464586..e77edeb 100644
--- a/example-setup/broker/configs/replication.config.template
+++ b/example-setup/broker/configs/replication.config.template
@@ -12,6 +12,7 @@
     maxApiPayloadSize = 40000
 [remote "$REMOTE"]
     url = http://$REMOTE_URL:8080/#{name}#.git
+    apiUrl = http://$REMOTE_URL:8080/
     fetch = +refs/*:refs/*
     mirror = true
     timeout = 60 # In seconds
diff --git a/example-setup/broker/entrypoint.sh b/example-setup/broker/entrypoint.sh
index 1d958e4..33df1d4 100755
--- a/example-setup/broker/entrypoint.sh
+++ b/example-setup/broker/entrypoint.sh
@@ -13,7 +13,7 @@
   cat /var/gerrit/etc/gerrit.config.template | envsubst > /var/gerrit/etc/gerrit.config
 }
 
-ARG JAVA_OPTS='--add-opens java.base/java.net=ALL-UNNAMED --add-opens java.base/java.lang.invoke=ALL-UNNAMED'
+JAVA_OPTS='--add-opens java.base/java.net=ALL-UNNAMED --add-opens java.base/java.lang.invoke=ALL-UNNAMED'
 
 echo "Init phase ..."
 java $JAVA_OPTS -jar /var/gerrit/bin/gerrit.war init --batch --install-all-plugins -d /var/gerrit
@@ -25,4 +25,4 @@
 java $JAVA_OPTS -jar /var/gerrit/bin/gerrit.war reindex -d /var/gerrit
 
 echo "Running Gerrit ..."
-exec /var/gerrit/bin/gerrit.sh run
\ No newline at end of file
+exec /var/gerrit/bin/gerrit.sh run
diff --git a/example-setup/http/entrypoint.sh b/example-setup/http/entrypoint.sh
index 1d958e4..33df1d4 100755
--- a/example-setup/http/entrypoint.sh
+++ b/example-setup/http/entrypoint.sh
@@ -13,7 +13,7 @@
   cat /var/gerrit/etc/gerrit.config.template | envsubst > /var/gerrit/etc/gerrit.config
 }
 
-ARG JAVA_OPTS='--add-opens java.base/java.net=ALL-UNNAMED --add-opens java.base/java.lang.invoke=ALL-UNNAMED'
+JAVA_OPTS='--add-opens java.base/java.net=ALL-UNNAMED --add-opens java.base/java.lang.invoke=ALL-UNNAMED'
 
 echo "Init phase ..."
 java $JAVA_OPTS -jar /var/gerrit/bin/gerrit.war init --batch --install-all-plugins -d /var/gerrit
@@ -25,4 +25,4 @@
 java $JAVA_OPTS -jar /var/gerrit/bin/gerrit.war reindex -d /var/gerrit
 
 echo "Running Gerrit ..."
-exec /var/gerrit/bin/gerrit.sh run
\ No newline at end of file
+exec /var/gerrit/bin/gerrit.sh run
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ApplyObjectCacheModule.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ApplyObjectCacheModule.java
new file mode 100644
index 0000000..944326e
--- /dev/null
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ApplyObjectCacheModule.java
@@ -0,0 +1,29 @@
+// Copyright (C) 2023 The Android Open Source Project
+//
+// Licensed 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 com.googlesource.gerrit.plugins.replication.pull;
+
+import com.google.gerrit.server.cache.CacheModule;
+import java.time.Duration;
+
+public class ApplyObjectCacheModule extends CacheModule {
+  public static final String APPLY_OBJECTS_CACHE = "apply_objects";
+  public static final Duration APPLY_OBJECTS_CACHE_MAX_AGE = Duration.ofMinutes(1);
+
+  @Override
+  protected void configure() {
+    cache(APPLY_OBJECTS_CACHE, ApplyObjectsCacheKey.class, Long.class)
+        .expireAfterWrite(APPLY_OBJECTS_CACHE_MAX_AGE);
+  }
+}
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ApplyObjectsCacheKey.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ApplyObjectsCacheKey.java
new file mode 100644
index 0000000..9e83b4d
--- /dev/null
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ApplyObjectsCacheKey.java
@@ -0,0 +1,31 @@
+// Copyright (C) 2023 The Android Open Source Project
+//
+// Licensed 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 com.googlesource.gerrit.plugins.replication.pull;
+
+import com.google.auto.value.AutoValue;
+
+@AutoValue
+public abstract class ApplyObjectsCacheKey {
+
+  public static ApplyObjectsCacheKey create(String objectId, String refName, String project) {
+    return new AutoValue_ApplyObjectsCacheKey(objectId, refName, project);
+  }
+
+  public abstract String objectId();
+
+  public abstract String refName();
+
+  public abstract String project();
+}
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationModule.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationModule.java
index 08a402e..d1c5ca1 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationModule.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationModule.java
@@ -19,12 +19,12 @@
 import com.google.common.eventbus.EventBus;
 import com.google.gerrit.extensions.annotations.Exports;
 import com.google.gerrit.extensions.config.CapabilityDefinition;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener;
 import com.google.gerrit.extensions.events.HeadUpdatedListener;
 import com.google.gerrit.extensions.events.LifecycleListener;
 import com.google.gerrit.extensions.events.ProjectDeletedListener;
 import com.google.gerrit.extensions.registration.DynamicSet;
 import com.google.gerrit.server.config.SitePaths;
+import com.google.gerrit.server.events.EventListener;
 import com.google.gerrit.server.events.EventTypes;
 import com.google.inject.AbstractModule;
 import com.google.inject.Inject;
@@ -80,6 +80,7 @@
     bind(RevisionReader.class).in(Scopes.SINGLETON);
     bind(ApplyObject.class);
     install(new FactoryModuleBuilder().build(FetchJob.Factory.class));
+    install(new ApplyObjectCacheModule());
     install(new PullReplicationApiModule());
 
     install(new FetchRefReplicatedEventModule());
@@ -123,7 +124,7 @@
         .annotatedWith(UniqueAnnotations.create())
         .to(ReplicationQueue.class);
 
-    DynamicSet.bind(binder(), GitBatchRefUpdateListener.class).to(ReplicationQueue.class);
+    DynamicSet.bind(binder(), EventListener.class).to(ReplicationQueue.class);
 
     bind(ConfigParser.class).to(SourceConfigParser.class).in(Scopes.SINGLETON);
 
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueue.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueue.java
index 990c82c..b88db69 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueue.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueue.java
@@ -20,13 +20,15 @@
 import com.google.gerrit.entities.Project;
 import com.google.gerrit.entities.Project.NameKey;
 import com.google.gerrit.entities.RefNames;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener;
 import com.google.gerrit.extensions.events.HeadUpdatedListener;
 import com.google.gerrit.extensions.events.LifecycleListener;
 import com.google.gerrit.extensions.events.ProjectDeletedListener;
 import com.google.gerrit.extensions.registration.DynamicItem;
 import com.google.gerrit.metrics.Timer1.Context;
+import com.google.gerrit.server.config.GerritInstanceId;
 import com.google.gerrit.server.events.EventDispatcher;
+import com.google.gerrit.server.events.EventListener;
+import com.google.gerrit.server.events.RefUpdatedEvent;
 import com.google.gerrit.server.git.WorkQueue;
 import com.google.inject.Inject;
 import com.google.inject.Provider;
@@ -36,6 +38,7 @@
 import com.googlesource.gerrit.plugins.replication.pull.api.exception.MissingParentObjectException;
 import com.googlesource.gerrit.plugins.replication.pull.client.FetchApiClient;
 import com.googlesource.gerrit.plugins.replication.pull.client.HttpResult;
+import com.googlesource.gerrit.plugins.replication.pull.filter.ApplyObjectsRefsFilter;
 import com.googlesource.gerrit.plugins.replication.pull.filter.ExcludedRefsFilter;
 import java.io.IOException;
 import java.net.URISyntaxException;
@@ -64,8 +67,8 @@
 
 public class ReplicationQueue
     implements ObservableQueue,
+        EventListener,
         LifecycleListener,
-        GitBatchRefUpdateListener,
         ProjectDeletedListener,
         HeadUpdatedListener {
 
@@ -73,6 +76,8 @@
   static final Logger repLog = LoggerFactory.getLogger(PULL_REPLICATION_LOG_NAME);
 
   private static final Integer DEFAULT_FETCH_CALLS_TIMEOUT = 0;
+  private static final String REF_UDPATED_EVENT_TYPE = new RefUpdatedEvent().type;
+  private static final String ZEROS_OBJECTID = ObjectId.zeroId().getName();
   private final ReplicationStateListener stateLog;
 
   private final WorkQueue workQueue;
@@ -87,6 +92,8 @@
   private Provider<RevisionReader> revReaderProvider;
   private final ApplyObjectMetrics applyObjectMetrics;
   private final FetchReplicationMetrics fetchMetrics;
+  private final String instanceId;
+  private ApplyObjectsRefsFilter applyObjectsRefsFilter;
 
   @Inject
   ReplicationQueue(
@@ -98,7 +105,9 @@
       ExcludedRefsFilter refsFilter,
       Provider<RevisionReader> revReaderProvider,
       ApplyObjectMetrics applyObjectMetrics,
-      FetchReplicationMetrics fetchMetrics) {
+      FetchReplicationMetrics fetchMetrics,
+      @GerritInstanceId String instanceId,
+      ApplyObjectsRefsFilter applyObjectsRefsFilter) {
     workQueue = wq;
     dispatcher = dis;
     sources = rd;
@@ -109,6 +118,8 @@
     this.revReaderProvider = revReaderProvider;
     this.applyObjectMetrics = applyObjectMetrics;
     this.fetchMetrics = fetchMetrics;
+    this.instanceId = instanceId;
+    this.applyObjectsRefsFilter = applyObjectsRefsFilter;
   }
 
   @Override
@@ -147,24 +158,21 @@
   }
 
   @Override
-  public void onGitBatchRefUpdate(GitBatchRefUpdateListener.Event event) {
-    event.getUpdatedRefs().stream()
-        .sorted(ReplicationQueue::sortByMetaRefAsLast)
-        .forEachOrdered(
-            updateRef -> {
-              String refName = updateRef.getRefName();
+  public void onEvent(com.google.gerrit.server.events.Event e) {
+    if (e.type.equals(REF_UDPATED_EVENT_TYPE) && instanceId.equals(e.instanceId)) {
+      RefUpdatedEvent event = (RefUpdatedEvent) e;
 
-              if (isRefToBeReplicated(refName)) {
-                repLog.info(
-                    "Ref event received: {} on project {}:{} - {} => {}",
-                    refUpdateType(updateRef),
-                    event.getProjectName(),
-                    refName,
-                    updateRef.getOldObjectId(),
-                    updateRef.getNewObjectId());
-                fire(ReferenceUpdatedEvent.from(event.getProjectName(), updateRef));
-              }
-            });
+      if (isRefToBeReplicated(event.getRefName())) {
+        repLog.info(
+            "Ref event received: {} on project {}:{} - {} => {}",
+            refUpdateType(event),
+            event.refUpdate.get().project,
+            event.getRefName(),
+            event.refUpdate.get().oldRev,
+            event.refUpdate.get().newRev);
+        fire(ReferenceUpdatedEvent.from(event));
+      }
+    }
   }
 
   @Override
@@ -177,20 +185,13 @@
                 source.getApis().forEach(apiUrl -> source.scheduleDeleteProject(apiUrl, project)));
   }
 
-  private static int sortByMetaRefAsLast(UpdatedRef a, @SuppressWarnings("unused") UpdatedRef b) {
-    repLog.info("sortByMetaRefAsLast(" + a.getRefName() + " <=> " + b.getRefName());
-    return Boolean.compare(
-        RefNames.isNoteDbMetaRef(a.getRefName()), RefNames.isNoteDbMetaRef(b.getRefName()));
-  }
-
-  private static String refUpdateType(UpdatedRef updateRef) {
-    String forcedPrefix = updateRef.isNonFastForward() ? "FORCED " : " ";
-    if (updateRef.isCreate()) {
-      return forcedPrefix + "CREATE";
-    } else if (updateRef.isDelete()) {
-      return forcedPrefix + "DELETE";
+  private static String refUpdateType(RefUpdatedEvent event) {
+    if (ZEROS_OBJECTID.equals(event.refUpdate.get().oldRev)) {
+      return "CREATE";
+    } else if (ZEROS_OBJECTID.equals(event.refUpdate.get().newRev)) {
+      return "DELETE";
     } else {
-      return forcedPrefix + "UPDATE";
+      return "UPDATE";
     }
   }
 
@@ -227,6 +228,7 @@
               Project.nameKey(event.projectName()),
               event.objectId(),
               event.refName(),
+              event.eventCreatedOn(),
               event.isDelete(),
               state);
       fetchCallsPool
@@ -250,9 +252,11 @@
       NameKey project,
       ObjectId objectId,
       String refName,
+      long eventCreatedOn,
       boolean isDelete,
       ReplicationState state) {
-    CallFunction call = getCallFunction(project, objectId, refName, isDelete, state);
+    CallFunction call =
+        getCallFunction(project, objectId, refName, eventCreatedOn, isDelete, state);
 
     return (source) -> {
       boolean callSuccessful;
@@ -277,10 +281,12 @@
       NameKey project,
       ObjectId objectId,
       String refName,
+      long eventCreatedOn,
       boolean isDelete,
       ReplicationState state) {
     if (isDelete) {
-      return ((source) -> callSendObject(source, project, refName, isDelete, null, state));
+      return ((source) ->
+          callSendObject(source, project, refName, eventCreatedOn, isDelete, null, state));
     }
 
     try {
@@ -295,7 +301,13 @@
       if (revisionData.isPresent()) {
         return ((source) ->
             callSendObject(
-                source, project, refName, isDelete, Arrays.asList(revisionData.get()), state));
+                source,
+                project,
+                refName,
+                eventCreatedOn,
+                isDelete,
+                Arrays.asList(revisionData.get()),
+                state));
       }
     } catch (InvalidObjectIdException | IOException e) {
       stateLog.error(
@@ -311,8 +323,9 @@
 
   private boolean callSendObject(
       Source source,
-      Project.NameKey project,
+      NameKey project,
       String refName,
+      long eventCreatedOn,
       boolean isDelete,
       List<RevisionData> revision,
       ReplicationState state)
@@ -332,8 +345,9 @@
           Context<String> apiTimer = applyObjectMetrics.startEnd2End(source.getRemoteConfigName());
           HttpResult result =
               isDelete
-                  ? fetchClient.callSendObject(project, refName, isDelete, null, uri)
-                  : fetchClient.callSendObjects(project, refName, revision, uri);
+                  ? fetchClient.callSendObject(
+                      project, refName, eventCreatedOn, isDelete, null, uri)
+                  : fetchClient.callSendObjects(project, refName, eventCreatedOn, revision, uri);
           boolean resultSuccessful = result.isSuccessful();
           repLog.info(
               "Pull replication REST API apply object to {} COMPLETED for {}:{} - {}, HTTP Result:"
@@ -355,7 +369,8 @@
           if (!resultSuccessful) {
             if (result.isParentObjectMissing()) {
 
-              if (RefNames.isNoteDbMetaRef(refName) && revision.size() == 1) {
+              if ((RefNames.isNoteDbMetaRef(refName) || applyObjectsRefsFilter.match(refName))
+                  && revision.size() == 1) {
                 List<RevisionData> allRevisions =
                     fetchWholeMetaHistory(project, refName, revision.get(0));
                 repLog.info(
@@ -364,7 +379,8 @@
                     project,
                     refName,
                     allRevisions);
-                return callSendObject(source, project, refName, isDelete, allRevisions, state);
+                return callSendObject(
+                    source, project, refName, eventCreatedOn, isDelete, allRevisions, state);
               }
 
               throw new MissingParentObjectException(
@@ -526,17 +542,22 @@
   abstract static class ReferenceUpdatedEvent {
 
     static ReferenceUpdatedEvent create(
-        String projectName, String refName, ObjectId objectId, boolean isDelete) {
+        String projectName,
+        String refName,
+        ObjectId objectId,
+        long eventCreatedOn,
+        boolean isDelete) {
       return new AutoValue_ReplicationQueue_ReferenceUpdatedEvent(
-          projectName, refName, objectId, isDelete);
+          projectName, refName, objectId, eventCreatedOn, isDelete);
     }
 
-    static ReferenceUpdatedEvent from(String projectName, UpdatedRef updateRef) {
+    static ReferenceUpdatedEvent from(RefUpdatedEvent event) {
       return ReferenceUpdatedEvent.create(
-          projectName,
-          updateRef.getRefName(),
-          ObjectId.fromString(updateRef.getNewObjectId()),
-          updateRef.isDelete());
+          event.refUpdate.get().project,
+          event.getRefName(),
+          ObjectId.fromString(event.refUpdate.get().newRev),
+          event.eventCreatedOn,
+          ZEROS_OBJECTID.equals(event.refUpdate.get().newRev));
     }
 
     public abstract String projectName();
@@ -545,6 +566,8 @@
 
     public abstract ObjectId objectId();
 
+    public abstract long eventCreatedOn();
+
     public abstract boolean isDelete();
   }
 
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/Source.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/Source.java
index ff1664e..0ebbf91 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/Source.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/Source.java
@@ -523,6 +523,12 @@
           pendingFetchOp.addStates(fetchOp.getStates());
           fetchOp.removeStates();
 
+          stateLog.warn(
+              String.format(
+                  "[%s] Merging all refs to fetch from [%s] to the already retrying task [%s] for keeping its position into the replication queue",
+                  fetchOp.getTaskIdHex(), fetchOp.getURI(), pendingFetchOp.getTaskIdHex()),
+              fetchOp.getStatesAsArray());
+
         } else {
           // The one pending is one that is NOT retrying, it was just
           // scheduled believing no problem would happen. The one pending
@@ -541,6 +547,12 @@
           fetchOp.addRefs(pendingFetchOp.getRefs());
           fetchOp.addStates(pendingFetchOp.getStates());
           pendingFetchOp.removeStates();
+
+          stateLog.warn(
+              String.format(
+                  "[%s] Merging the pending fetch from [%s] with task [%s] and rescheduling",
+                  pendingFetchOp.getTaskIdHex(), pendingFetchOp.getURI(), fetchOp.getTaskIdHex()),
+              pendingFetchOp.getStatesAsArray());
         }
       }
 
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectAction.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectAction.java
index 04cc9e8..b3379f2 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectAction.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectAction.java
@@ -97,7 +97,11 @@
       }
 
       applyObjectCommand.applyObject(
-          resource.getNameKey(), input.getRefName(), input.getRevisionData(), input.getLabel());
+          resource.getNameKey(),
+          input.getRefName(),
+          input.getRevisionData(),
+          input.getLabel(),
+          input.getEventCreatedOn());
       return Response.created(input);
     } catch (MissingParentObjectException e) {
       repLog.error(
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommand.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommand.java
index c268ba1..968a03c 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommand.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommand.java
@@ -14,9 +14,11 @@
 
 package com.googlesource.gerrit.plugins.replication.pull.api;
 
+import static com.googlesource.gerrit.plugins.replication.pull.ApplyObjectCacheModule.APPLY_OBJECTS_CACHE;
 import static com.googlesource.gerrit.plugins.replication.pull.PullReplicationLogger.repLog;
 import static java.util.concurrent.TimeUnit.NANOSECONDS;
 
+import com.google.common.cache.Cache;
 import com.google.common.collect.ImmutableSet;
 import com.google.common.flogger.FluentLogger;
 import com.google.gerrit.entities.Project;
@@ -25,21 +27,20 @@
 import com.google.gerrit.server.events.EventDispatcher;
 import com.google.gerrit.server.permissions.PermissionBackendException;
 import com.google.inject.Inject;
-import com.googlesource.gerrit.plugins.replication.pull.ApplyObjectMetrics;
-import com.googlesource.gerrit.plugins.replication.pull.Context;
-import com.googlesource.gerrit.plugins.replication.pull.FetchRefReplicatedEvent;
-import com.googlesource.gerrit.plugins.replication.pull.PullReplicationStateLogger;
-import com.googlesource.gerrit.plugins.replication.pull.ReplicationState;
+import com.google.inject.name.Named;
+import com.googlesource.gerrit.plugins.replication.pull.*;
 import com.googlesource.gerrit.plugins.replication.pull.ReplicationState.RefFetchResult;
 import com.googlesource.gerrit.plugins.replication.pull.Source;
 import com.googlesource.gerrit.plugins.replication.pull.SourcesCollection;
 import com.googlesource.gerrit.plugins.replication.pull.api.data.RevisionData;
+import com.googlesource.gerrit.plugins.replication.pull.api.data.RevisionObjectData;
 import com.googlesource.gerrit.plugins.replication.pull.api.exception.MissingParentObjectException;
 import com.googlesource.gerrit.plugins.replication.pull.api.exception.RefUpdateException;
 import com.googlesource.gerrit.plugins.replication.pull.fetch.ApplyObject;
 import com.googlesource.gerrit.plugins.replication.pull.fetch.RefUpdateState;
 import java.io.IOException;
 import java.util.Arrays;
+import java.util.List;
 import java.util.Set;
 import org.eclipse.jgit.lib.RefUpdate;
 import org.eclipse.jgit.transport.RefSpec;
@@ -47,7 +48,7 @@
 public class ApplyObjectCommand {
 
   private static final FluentLogger logger = FluentLogger.forEnclosingClass();
-
+  private final Cache<ApplyObjectsCacheKey, Long> refUpdatesSucceededCache;
   private static final Set<RefUpdate.Result> SUCCESSFUL_RESULTS =
       ImmutableSet.of(
           RefUpdate.Result.NEW,
@@ -67,22 +68,32 @@
       ApplyObject applyObject,
       ApplyObjectMetrics metrics,
       DynamicItem<EventDispatcher> eventDispatcher,
-      SourcesCollection sourcesCollection) {
+      SourcesCollection sourcesCollection,
+      @Named(APPLY_OBJECTS_CACHE) Cache<ApplyObjectsCacheKey, Long> refUpdatesSucceededCache) {
     this.fetchStateLog = fetchStateLog;
     this.applyObject = applyObject;
     this.metrics = metrics;
     this.eventDispatcher = eventDispatcher;
     this.sourcesCollection = sourcesCollection;
+    this.refUpdatesSucceededCache = refUpdatesSucceededCache;
   }
 
   public void applyObject(
-      Project.NameKey name, String refName, RevisionData revisionsData, String sourceLabel)
+      Project.NameKey name,
+      String refName,
+      RevisionData revisionsData,
+      String sourceLabel,
+      long eventCreatedOn)
       throws IOException, RefUpdateException, MissingParentObjectException {
-    applyObjects(name, refName, new RevisionData[] {revisionsData}, sourceLabel);
+    applyObjects(name, refName, new RevisionData[] {revisionsData}, sourceLabel, eventCreatedOn);
   }
 
   public void applyObjects(
-      Project.NameKey name, String refName, RevisionData[] revisionsData, String sourceLabel)
+      Project.NameKey name,
+      String refName,
+      RevisionData[] revisionsData,
+      String sourceLabel,
+      long eventCreatedOn)
       throws IOException, RefUpdateException, MissingParentObjectException {
 
     repLog.info(
@@ -94,7 +105,26 @@
     Timer1.Context<String> context = metrics.start(sourceLabel);
 
     RefUpdateState refUpdateState = applyObject.apply(name, new RefSpec(refName), revisionsData);
+    Boolean isRefUpdateSuccessful = isSuccessful(refUpdateState.getResult());
 
+    if (isRefUpdateSuccessful) {
+      for (RevisionData revisionData : revisionsData) {
+        RevisionObjectData commitObj = revisionData.getCommitObject();
+        List<RevisionObjectData> blobs = revisionData.getBlobs();
+
+        if (commitObj != null) {
+          refUpdatesSucceededCache.put(
+              ApplyObjectsCacheKey.create(
+                  revisionData.getCommitObject().getSha1(), refName, name.get()),
+              eventCreatedOn);
+        } else if (blobs != null) {
+          for (RevisionObjectData blob : blobs) {
+            refUpdatesSucceededCache.put(
+                ApplyObjectsCacheKey.create(blob.getSha1(), refName, name.get()), eventCreatedOn);
+          }
+        }
+      }
+    }
     long elapsed = NANOSECONDS.toMillis(context.stop());
 
     try {
@@ -122,7 +152,7 @@
       Context.unsetLocalEvent();
     }
 
-    if (!isSuccessful(refUpdateState.getResult())) {
+    if (!isRefUpdateSuccessful) {
       String message =
           String.format(
               "RefUpdate failed with result %s for: sourceLcabel=%s, project=%s, refName=%s",
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectsAction.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectsAction.java
index a1e1f5b..dd155d4 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectsAction.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectsAction.java
@@ -97,7 +97,11 @@
       }
 
       command.applyObjects(
-          resource.getNameKey(), input.getRefName(), input.getRevisionsData(), input.getLabel());
+          resource.getNameKey(),
+          input.getRefName(),
+          input.getRevisionsData(),
+          input.getLabel(),
+          input.getEventCreatedOn());
       return Response.created(input);
     } catch (MissingParentObjectException e) {
       repLog.error(
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/PullReplicationEndpoints.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/PullReplicationEndpoints.java
new file mode 100644
index 0000000..fc97945
--- /dev/null
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/PullReplicationEndpoints.java
@@ -0,0 +1,39 @@
+// Copyright (C) 2023 The Android Open Source Project
+//
+// Licensed 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 com.googlesource.gerrit.plugins.replication.pull.api;
+
+import static com.google.gerrit.common.UsedAt.Project.PLUGIN_MULTI_SITE;
+
+import com.google.gerrit.common.UsedAt;
+
+/**
+ * Temporary solution for stable branches for allowing the multi-site plugin to understand his
+ * caller identity.
+ *
+ * <p>TODO: To be removed from v3.9/master, where this problem does not exist anymore because of a
+ * clearer definition of responsibilities between the multi-site and the pull-replication plugins.
+ */
+public interface PullReplicationEndpoints {
+
+  @UsedAt(PLUGIN_MULTI_SITE)
+  public static final String APPLY_OBJECT_API_ENDPOINT = "apply-object";
+
+  @UsedAt(PLUGIN_MULTI_SITE)
+  public static final String APPLY_OBJECTS_API_ENDPOINT = "apply-objects";
+
+  public static final String FETCH_ENDPOINT = "fetch";
+  public static final String INIT_PROJECT_ENDPOINT = "init-project";
+  public static final String DELETE_PROJECT_ENDPOINT = "delete-project";
+}
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/PullReplicationFilter.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/PullReplicationFilter.java
index 7027239..855c0cf 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/PullReplicationFilter.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/PullReplicationFilter.java
@@ -56,7 +56,6 @@
 import java.io.EOFException;
 import java.io.IOException;
 import java.io.PrintWriter;
-import java.nio.file.InvalidPathException;
 import java.util.Map;
 import java.util.NoSuchElementException;
 import java.util.Optional;
@@ -69,7 +68,7 @@
 import javax.servlet.http.HttpServletRequest;
 import javax.servlet.http.HttpServletResponse;
 
-public class PullReplicationFilter extends AllRequestFilter {
+public class PullReplicationFilter extends AllRequestFilter implements PullReplicationEndpoints {
   private static final FluentLogger logger = FluentLogger.forEnclosingClass();
 
   private static final Pattern projectNameInGerritUrl = Pattern.compile(".*/projects/([^/]+)/.*");
@@ -159,7 +158,8 @@
       RestApiServlet.replyError(
           httpRequest, httpResponse, SC_BAD_REQUEST, "Project name not present in the url", e);
     } catch (Exception e) {
-      if (e instanceof InvalidPathException || e.getCause() instanceof InvalidPathException) {
+      if (e instanceof IllegalArgumentException
+          || e.getCause() instanceof IllegalArgumentException) {
         RestApiServlet.replyError(
             httpRequest, httpResponse, SC_BAD_REQUEST, "Invalid repository path in request", e);
       } else {
@@ -299,19 +299,25 @@
   }
 
   private boolean isApplyObjectAction(HttpServletRequest httpRequest) {
-    return httpRequest.getRequestURI().endsWith(String.format("/%s~apply-object", pluginName));
+    return httpRequest
+        .getRequestURI()
+        .endsWith(String.format("/%s~" + APPLY_OBJECT_API_ENDPOINT, pluginName));
   }
 
   private boolean isApplyObjectsAction(HttpServletRequest httpRequest) {
-    return httpRequest.getRequestURI().endsWith(String.format("/%s~apply-objects", pluginName));
+    return httpRequest
+        .getRequestURI()
+        .endsWith(String.format("/%s~" + APPLY_OBJECTS_API_ENDPOINT, pluginName));
   }
 
   private boolean isFetchAction(HttpServletRequest httpRequest) {
-    return httpRequest.getRequestURI().endsWith(String.format("/%s~fetch", pluginName));
+    return httpRequest.getRequestURI().endsWith(String.format("/%s~" + FETCH_ENDPOINT, pluginName));
   }
 
   private boolean isInitProjectAction(HttpServletRequest httpRequest) {
-    return httpRequest.getRequestURI().contains(String.format("/%s/init-project/", pluginName));
+    return httpRequest
+        .getRequestURI()
+        .contains(String.format("/%s/" + INIT_PROJECT_ENDPOINT + "/", pluginName));
   }
 
   private boolean isUpdateHEADAction(HttpServletRequest httpRequest) {
@@ -320,7 +326,9 @@
   }
 
   private boolean isDeleteProjectAction(HttpServletRequest httpRequest) {
-    return httpRequest.getRequestURI().endsWith(String.format("/%s~delete-project", pluginName))
+    return httpRequest
+            .getRequestURI()
+            .endsWith(String.format("/%s~" + DELETE_PROJECT_ENDPOINT, pluginName))
         && "DELETE".equals(httpRequest.getMethod());
   }
 }
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionInput.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionInput.java
index c18e11d..6883f2c 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionInput.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionInput.java
@@ -23,11 +23,14 @@
 
   private String refName;
 
+  private long eventCreatedOn;
   private RevisionData revisionData;
 
-  public RevisionInput(String label, String refName, RevisionData revisionData) {
+  public RevisionInput(
+      String label, String refName, long eventCreatedOn, RevisionData revisionData) {
     this.label = label;
     this.refName = refName;
+    this.eventCreatedOn = eventCreatedOn;
     this.revisionData = revisionData;
   }
 
@@ -43,6 +46,10 @@
     return revisionData;
   }
 
+  public long getEventCreatedOn() {
+    return eventCreatedOn;
+  }
+
   public void validate() {
     validate(refName, revisionData);
   }
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionsInput.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionsInput.java
index 2361f6b..d4626f4 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionsInput.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/data/RevisionsInput.java
@@ -21,11 +21,14 @@
 
   private String refName;
 
+  private long eventCreatedOn;
   private RevisionData[] revisionsData;
 
-  public RevisionsInput(String label, String refName, RevisionData[] revisionsData) {
+  public RevisionsInput(
+      String label, String refName, long eventCreatedOn, RevisionData[] revisionsData) {
     this.label = label;
     this.refName = refName;
+    this.eventCreatedOn = eventCreatedOn;
     this.revisionsData = revisionsData;
   }
 
@@ -37,6 +40,10 @@
     return refName;
   }
 
+  public long getEventCreatedOn() {
+    return eventCreatedOn;
+  }
+
   public RevisionData[] getRevisionsData() {
     return revisionsData;
   }
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/auth/PullReplicationGroupBackend.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/auth/PullReplicationGroupBackend.java
index ffd3c1f..c61d4d9 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/auth/PullReplicationGroupBackend.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/auth/PullReplicationGroupBackend.java
@@ -28,7 +28,6 @@
 import java.util.Arrays;
 import java.util.Collection;
 import java.util.Collections;
-import java.util.List;
 
 /** Backend to expose the pull-replication internal user group membership. */
 @Singleton
@@ -84,7 +83,7 @@
   @Override
   public Collection<GroupReference> suggest(String name, ProjectState project) {
     return NAME_PREFIX.contains(name.toLowerCase())
-        ? List.of(GroupReference.create(INTERNAL_GROUP_UUID, INTERNAL_GROUP_NAME))
+        ? Arrays.asList(GroupReference.create(INTERNAL_GROUP_UUID, INTERNAL_GROUP_NAME))
         : Collections.emptyList();
   }
 
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchApiClient.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchApiClient.java
index 6000eb9..1991260 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchApiClient.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchApiClient.java
@@ -17,6 +17,7 @@
 import static java.util.concurrent.TimeUnit.MILLISECONDS;
 
 import com.google.gerrit.entities.Project;
+import com.google.gerrit.entities.Project.NameKey;
 import com.googlesource.gerrit.plugins.replication.pull.Source;
 import com.googlesource.gerrit.plugins.replication.pull.api.data.RevisionData;
 import java.io.IOException;
@@ -46,14 +47,19 @@
   HttpResult updateHead(Project.NameKey project, String newHead, URIish apiUri) throws IOException;
 
   HttpResult callSendObject(
-      Project.NameKey project,
+      NameKey project,
       String refName,
+      long eventCreatedOn,
       boolean isDelete,
       RevisionData revisionData,
       URIish targetUri)
       throws ClientProtocolException, IOException;
 
   HttpResult callSendObjects(
-      Project.NameKey project, String refName, List<RevisionData> revisionData, URIish targetUri)
+      NameKey project,
+      String refName,
+      long eventCreatedOn,
+      List<RevisionData> revisionData,
+      URIish targetUri)
       throws ClientProtocolException, IOException;
 }
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClient.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClient.java
index cbc2cf7..b606ba8 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClient.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClient.java
@@ -171,8 +171,9 @@
    */
   @Override
   public HttpResult callSendObject(
-      Project.NameKey project,
+      NameKey project,
       String refName,
+      long eventCreatedOn,
       boolean isDelete,
       @Nullable RevisionData revisionData,
       URIish targetUri)
@@ -184,7 +185,7 @@
     } else {
       requireNull(revisionData, "DELETE ref-updates cannot be associated with a RevisionData");
     }
-    RevisionInput input = new RevisionInput(instanceId, refName, revisionData);
+    RevisionInput input = new RevisionInput(instanceId, refName, eventCreatedOn, revisionData);
 
     String url = formatUrl(targetUri.toString(), project, "apply-object");
 
@@ -196,14 +197,20 @@
 
   @Override
   public HttpResult callSendObjects(
-      NameKey project, String refName, List<RevisionData> revisionData, URIish targetUri)
+      NameKey project,
+      String refName,
+      long eventCreatedOn,
+      List<RevisionData> revisionData,
+      URIish targetUri)
       throws ClientProtocolException, IOException {
     if (revisionData.size() == 1) {
-      return callSendObject(project, refName, false, revisionData.get(0), targetUri);
+      return callSendObject(
+          project, refName, eventCreatedOn, false, revisionData.get(0), targetUri);
     }
 
     RevisionData[] inputData = new RevisionData[revisionData.size()];
-    RevisionsInput input = new RevisionsInput(instanceId, refName, revisionData.toArray(inputData));
+    RevisionsInput input =
+        new RevisionsInput(instanceId, refName, eventCreatedOn, revisionData.toArray(inputData));
 
     String url = formatUrl(targetUri.toString(), project, "apply-objects");
     HttpPost post = new HttpPost(url);
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListener.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListener.java
index 2ea8a33..b14d4a1 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListener.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListener.java
@@ -14,9 +14,11 @@
 
 package com.googlesource.gerrit.plugins.replication.pull.event;
 
+import static com.googlesource.gerrit.plugins.replication.pull.ApplyObjectCacheModule.APPLY_OBJECTS_CACHE;
 import static java.util.Objects.requireNonNull;
 
 import com.google.common.base.Strings;
+import com.google.common.cache.Cache;
 import com.google.common.flogger.FluentLogger;
 import com.google.gerrit.common.Nullable;
 import com.google.gerrit.entities.Project.NameKey;
@@ -24,6 +26,7 @@
 import com.google.gerrit.extensions.restapi.AuthException;
 import com.google.gerrit.extensions.restapi.RestApiException;
 import com.google.gerrit.server.config.GerritInstanceId;
+import com.google.gerrit.server.data.RefUpdateAttribute;
 import com.google.gerrit.server.events.Event;
 import com.google.gerrit.server.events.EventListener;
 import com.google.gerrit.server.events.ProjectCreatedEvent;
@@ -33,6 +36,8 @@
 import com.google.gerrit.server.permissions.PermissionBackendException;
 import com.google.inject.Inject;
 import com.google.inject.Provider;
+import com.google.inject.name.Named;
+import com.googlesource.gerrit.plugins.replication.pull.ApplyObjectsCacheKey;
 import com.googlesource.gerrit.plugins.replication.pull.FetchOne;
 import com.googlesource.gerrit.plugins.replication.pull.Source;
 import com.googlesource.gerrit.plugins.replication.pull.SourcesCollection;
@@ -59,6 +64,7 @@
   private final SourcesCollection sources;
   private final String instanceId;
   private final WorkQueue workQueue;
+  private final Cache<ApplyObjectsCacheKey, Long> refUpdatesSucceededCache;
 
   @Inject
   public StreamEventListener(
@@ -69,7 +75,8 @@
       FetchJob.Factory fetchJobFactory,
       Provider<PullReplicationApiRequestMetrics> metricsProvider,
       SourcesCollection sources,
-      ExcludedRefsFilter excludedRefsFilter) {
+      ExcludedRefsFilter excludedRefsFilter,
+      @Named(APPLY_OBJECTS_CACHE) Cache<ApplyObjectsCacheKey, Long> refUpdatesSucceededCache) {
     this.instanceId = instanceId;
     this.deleteCommand = deleteCommand;
     this.projectInitializationAction = projectInitializationAction;
@@ -78,6 +85,7 @@
     this.metricsProvider = metricsProvider;
     this.sources = sources;
     this.refsFilter = excludedRefsFilter;
+    this.refUpdatesSucceededCache = refUpdatesSucceededCache;
 
     requireNonNull(
         Strings.emptyToNull(this.instanceId), "gerrit.instanceId cannot be null or empty");
@@ -120,6 +128,17 @@
         return;
       }
 
+      if (isApplyObjectsCacheHit(refUpdatedEvent)) {
+        logger.atFine().log(
+            "Skipping refupdate '%s'  '%s'=>'%s' (eventCreatedOn=%d) for project '%s' because has been already replicated via apply-object",
+            refUpdatedEvent.getRefName(),
+            refUpdatedEvent.refUpdate.get().oldRev,
+            refUpdatedEvent.refUpdate.get().newRev,
+            refUpdatedEvent.eventCreatedOn,
+            refUpdatedEvent.getProjectNameKey());
+        return;
+      }
+
       fetchRefsAsync(
           refUpdatedEvent.getRefName(),
           refUpdatedEvent.instanceId,
@@ -207,4 +226,15 @@
   private String getProjectRepositoryName(ProjectCreatedEvent projectCreatedEvent) {
     return String.format("%s.git", projectCreatedEvent.projectName);
   }
+
+  private boolean isApplyObjectsCacheHit(RefUpdatedEvent refUpdateEvent) {
+    RefUpdateAttribute refUpdateAttribute = refUpdateEvent.refUpdate.get();
+    Long refUpdateSuccededTimestamp =
+        refUpdatesSucceededCache.getIfPresent(
+            ApplyObjectsCacheKey.create(
+                refUpdateAttribute.newRev, refUpdateAttribute.refName, refUpdateAttribute.project));
+
+    return refUpdateSuccededTimestamp != null
+        && refUpdateEvent.eventCreatedOn <= refUpdateSuccededTimestamp;
+  }
 }
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/InexistentRefTransportException.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/InexistentRefTransportException.java
index 7c2e637..f97f648 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/InexistentRefTransportException.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/InexistentRefTransportException.java
@@ -40,8 +40,11 @@
   }
 
   public static Optional<TransportException> getOptionalPermanentFailure(TransportException e) {
-    return wrapException(JGIT_INEXISTENT_REF_PATTERN, e)
-        .or(() -> wrapException(CGIT_INEXISTENT_REF_PATTERN, e));
+    Optional<TransportException> jgitEx = wrapException(JGIT_INEXISTENT_REF_PATTERN, e);
+    if (jgitEx.isPresent()) {
+      return jgitEx;
+    }
+    return wrapException(CGIT_INEXISTENT_REF_PATTERN, e);
   }
 
   private static Optional<TransportException> wrapException(
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/filter/ApplyObjectsRefsFilter.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/filter/ApplyObjectsRefsFilter.java
new file mode 100644
index 0000000..ff5899b
--- /dev/null
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/filter/ApplyObjectsRefsFilter.java
@@ -0,0 +1,37 @@
+// Copyright (C) 2023 The Android Open Source Project
+//
+// Licensed 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 com.googlesource.gerrit.plugins.replication.pull.filter;
+
+import com.google.common.collect.ImmutableList;
+import com.google.inject.Inject;
+import com.google.inject.Singleton;
+import com.googlesource.gerrit.plugins.replication.ReplicationConfig;
+import java.util.List;
+import org.eclipse.jgit.lib.Config;
+
+@Singleton
+public class ApplyObjectsRefsFilter extends RefsFilter {
+  @Inject
+  public ApplyObjectsRefsFilter(ReplicationConfig replicationConfig) {
+    super(replicationConfig);
+  }
+
+  @Override
+  protected List<String> getRefNamePatterns(Config cfg) {
+    String[] applyObjectsRefs =
+        cfg.getStringList("replication", null, "fallbackToApplyObjectsRefs");
+    return ImmutableList.copyOf(applyObjectsRefs);
+  }
+}
diff --git a/src/main/resources/Documentation/config.md b/src/main/resources/Documentation/config.md
index eab5fe6..b85aeb6 100644
--- a/src/main/resources/Documentation/config.md
+++ b/src/main/resources/Documentation/config.md
@@ -49,6 +49,37 @@
 To manually trigger replication at runtime, see
 SSH command [start](cmd-start.md).
 
+File `gerrit.config`
+-------------------------
+
+cache.@PLUGIN@-apply_objects.maxAge
+:	Maximum age to keep history of the latest successful apply-object refs.
+	Values should use common unit suffixes to express their setting:
+
+	s, sec, second, seconds
+
+	m, min, minute, minutes
+
+	h, hr, hour, hours
+
+	d, day, days
+
+	w, week, weeks (1 week is treated as 7 days)
+
+	mon, month, months (1 month is treated as 30 days)
+
+	y, year, years (1 year is treated as 365 days)
+
+	If a unit suffix is not specified, seconds is assumed. If 0 is supplied, the maximum age
+	is infinite and items are never purged except when the cache is full.
+
+	Default is 60s.
+
+cache.@PLUGIN@-apply_objects.memoryLimit
+:	The maximum number of apply-object refs retained in memory.
+
+	Default is 1024.
+
 File `@PLUGIN@.config`
 -------------------------
 
@@ -261,6 +292,33 @@
 
     By default, set to '*' (all refs are replicated synchronously).
 
+replication.fallbackToApplyObjectsRefs
+:   Specify for which refs will fallback to apply-objects REST-API that is capable of applying
+an entire chain of commits on the ref chain so that the Git fetch can be avoided altogether.
+It can be provided more than once, and supports three formats: regular expressions,
+wildcard matching, and single ref matching. All three formats matches are case-sensitive.
+
+    Values starting with a caret `^` are treated as regular
+    expressions. For the regular expressions details please follow
+    official [java documentation](https://docs.oracle.com/javase/tutorial/essential/regex/).
+
+    Please note that regular expressions could also be used
+    with inverse match.
+
+    Values that are not regular expressions and end in `*` are
+    treated as wildcard matches. Wildcards match refs whose
+    name agrees from the beginning until the trailing `*`. So
+    `foo/b*` would match the refs `foo/b`, `foo/bar`, and
+    `foo/baz`, but neither `foobar`, nor `bar/foo/baz`.
+
+    Values that are neither regular expressions nor wildcards are
+    treated as single ref matches. So `foo/bar` matches only
+    the ref `foo/bar`, but no other refs.
+
+    By default, set to 'refs/changes/**/meta'.
+
+    Note that the refs/changes/**/meta always fallback to apply-objects REST-API
+
 replication.maxApiPayloadSize
 :	Maximum size in bytes of the ref to be sent as a REST Api call
 	payload. For refs larger than threshold git fetch operation
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FakeGitReferenceUpdatedEvent.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FakeGitReferenceUpdatedEvent.java
index 69549aa..1472be2 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FakeGitReferenceUpdatedEvent.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FakeGitReferenceUpdatedEvent.java
@@ -14,96 +14,24 @@
 
 package com.googlesource.gerrit.plugins.replication.pull;
 
+import com.google.common.base.Suppliers;
 import com.google.gerrit.entities.Project;
-import com.google.gerrit.extensions.api.changes.NotifyHandling;
-import com.google.gerrit.extensions.common.AccountInfo;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener.UpdatedRef;
-import java.util.Set;
-import org.eclipse.jgit.lib.ObjectId;
-import org.eclipse.jgit.transport.ReceiveCommand;
+import com.google.gerrit.server.data.RefUpdateAttribute;
+import com.google.gerrit.server.events.RefUpdatedEvent;
 
-public class FakeGitReferenceUpdatedEvent implements GitBatchRefUpdateListener.Event {
-  private final String projectName;
-  private final String ref;
-  private final String oldObjectId;
-  private final String newObjectId;
-  private final ReceiveCommand.Type type;
-
+public class FakeGitReferenceUpdatedEvent extends RefUpdatedEvent {
   FakeGitReferenceUpdatedEvent(
       Project.NameKey project,
       String ref,
       String oldObjectId,
       String newObjectId,
-      ReceiveCommand.Type type) {
-    this.projectName = project.get();
-    this.ref = ref;
-    this.oldObjectId = oldObjectId;
-    this.newObjectId = newObjectId;
-    this.type = type;
-  }
-
-  @Override
-  public String getProjectName() {
-    return projectName;
-  }
-
-  @Override
-  public AccountInfo getUpdater() {
-    return null;
-  }
-
-  @Override
-  public String toString() {
-    return String.format(
-        "%s[%s,%s: %s -> %s]",
-        getClass().getSimpleName(), projectName, ref, oldObjectId, newObjectId);
-  }
-
-  @Override
-  public NotifyHandling getNotify() {
-    return NotifyHandling.ALL;
-  }
-
-  @Override
-  public Set<UpdatedRef> getUpdatedRefs() {
-    return Set.of(
-        new GitBatchRefUpdateListener.UpdatedRef() {
-
-          @Override
-          public String getRefName() {
-            return ref;
-          }
-
-          @Override
-          public String getOldObjectId() {
-            return ObjectId.zeroId().getName();
-          }
-
-          @Override
-          public String getNewObjectId() {
-            return newObjectId;
-          }
-
-          @Override
-          public boolean isCreate() {
-            return type == ReceiveCommand.Type.CREATE;
-          }
-
-          @Override
-          public boolean isDelete() {
-            return type == ReceiveCommand.Type.DELETE;
-          }
-
-          @Override
-          public boolean isNonFastForward() {
-            return type == ReceiveCommand.Type.UPDATE_NONFASTFORWARD;
-          }
-        });
-  }
-
-  @Override
-  public Set<String> getRefNames() {
-    return Set.of(ref);
+      String instanceId) {
+    RefUpdateAttribute upd = new RefUpdateAttribute();
+    upd.newRev = newObjectId;
+    upd.oldRev = oldObjectId;
+    upd.project = project.get();
+    upd.refName = ref;
+    this.refUpdate = Suppliers.ofInstance(upd);
+    this.instanceId = instanceId;
   }
 }
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationAsyncIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationAsyncIT.java
index 58be58e..be017d0 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationAsyncIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationAsyncIT.java
@@ -28,7 +28,7 @@
     name = "pull-replication",
     sysModule = "com.googlesource.gerrit.plugins.replication.pull.PullReplicationModule",
     httpModule = "com.googlesource.gerrit.plugins.replication.pull.api.HttpModule")
-public class PullReplicationAsyncIT extends PullReplicationIT {
+public class PullReplicationAsyncIT extends PullReplicationITAbstract {
   @Inject private SitePaths sitePaths;
 
   @Override
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationFanoutConfigIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationFanoutConfigIT.java
index ff8265f..b9ea0c8 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationFanoutConfigIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationFanoutConfigIT.java
@@ -27,7 +27,6 @@
 import com.google.gerrit.acceptance.testsuite.project.ProjectOperations;
 import com.google.gerrit.entities.Project;
 import com.google.gerrit.extensions.api.projects.BranchInput;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener;
 import com.google.gerrit.server.config.SitePaths;
 import com.google.inject.Inject;
 import com.googlesource.gerrit.plugins.replication.AutoReloadConfigDecorator;
@@ -44,7 +43,6 @@
 import org.eclipse.jgit.lib.Repository;
 import org.eclipse.jgit.revwalk.RevCommit;
 import org.eclipse.jgit.storage.file.FileBasedConfig;
-import org.eclipse.jgit.transport.ReceiveCommand;
 import org.eclipse.jgit.util.FS;
 import org.junit.Test;
 
@@ -105,14 +103,14 @@
     String sourceRef = pushResult.getPatchSet().refName();
 
     ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             sourceRef,
             ObjectId.zeroId().getName(),
             sourceCommit.getId().getName(),
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project)) {
       waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
@@ -141,14 +139,14 @@
     RevCommit sourceCommit = pushResult.getCommit();
     final String sourceRef = pushResult.getPatchSet().refName();
     ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             sourceRef,
             ObjectId.zeroId().getName(),
             sourceCommit.getId().getName(),
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project)) {
       waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
@@ -174,14 +172,14 @@
 
     ReplicationQueue pullReplicationQueue =
         plugin.getSysInjector().getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             newBranch,
             ObjectId.zeroId().getName(),
             branchRevision,
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project);
         Repository sourceRepo = repoManager.openRepository(project)) {
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationIT.java
index 6fd5cb3..d1aaf7c 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationIT.java
@@ -30,7 +30,6 @@
 import com.google.gerrit.entities.RefNames;
 import com.google.gerrit.extensions.api.changes.NotifyHandling;
 import com.google.gerrit.extensions.api.projects.BranchInput;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener;
 import com.google.gerrit.extensions.events.HeadUpdatedListener;
 import com.google.gerrit.extensions.events.ProjectDeletedListener;
 import com.google.gerrit.extensions.restapi.RestApiException;
@@ -47,7 +46,6 @@
 import org.eclipse.jgit.lib.Repository;
 import org.eclipse.jgit.revwalk.RevCommit;
 import org.eclipse.jgit.transport.PushResult;
-import org.eclipse.jgit.transport.ReceiveCommand;
 import org.eclipse.jgit.transport.RemoteRefUpdate;
 import org.eclipse.jgit.transport.RemoteRefUpdate.Status;
 import org.junit.Test;
@@ -91,14 +89,14 @@
     String sourceRef = pushResult.getPatchSet().refName();
 
     ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             sourceRef,
             ObjectId.zeroId().getName(),
             sourceCommit.getId().getName(),
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project)) {
       waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
@@ -124,14 +122,14 @@
 
     ReplicationQueue pullReplicationQueue =
         plugin.getSysInjector().getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             newBranch,
             ObjectId.zeroId().getName(),
             branchRevision,
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project);
         Repository sourceRepo = repoManager.openRepository(project)) {
@@ -167,14 +165,14 @@
 
     ReplicationQueue pullReplicationQueue =
         plugin.getSysInjector().getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             newBranch,
             ObjectId.zeroId().getName(),
             branchRevision,
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project)) {
       waitUntil(() -> checkedGetRef(repo, newBranch) != null);
@@ -193,14 +191,14 @@
     assertThat(pushedRefs).hasSize(1);
     assertThat(pushedRefs.iterator().next().getStatus()).isEqualTo(Status.OK);
 
-    GitBatchRefUpdateListener.Event forcedPushEvent =
+    FakeGitReferenceUpdatedEvent forcedPushEvent =
         new FakeGitReferenceUpdatedEvent(
             project,
             newBranch,
             branchRevision,
             amendedCommit.getId().getName(),
-            ReceiveCommand.Type.UPDATE_NONFASTFORWARD);
-    pullReplicationQueue.onGitBatchRefUpdate(forcedPushEvent);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(forcedPushEvent);
 
     try (Repository repo = repoManager.openRepository(project);
         Repository sourceRepo = repoManager.openRepository(project)) {
@@ -232,14 +230,14 @@
     String sourceRef = pushResult.getPatchSet().refName();
 
     ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             sourceRef,
             ObjectId.zeroId().getName(),
             sourceCommit.getId().getName(),
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project)) {
       waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
@@ -273,14 +271,14 @@
 
     ReplicationQueue pullReplicationQueue =
         plugin.getSysInjector().getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             newBranch,
             ObjectId.zeroId().getName(),
             branchRevision,
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project);
         Repository sourceRepo = repoManager.openRepository(project)) {
@@ -364,14 +362,14 @@
     String sourceRef = pushResult.getPatchSet().refName();
 
     ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             sourceRef,
             ObjectId.zeroId().getName(),
             sourceCommit.getId().getName(),
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project)) {
       waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationITAbstract.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationITAbstract.java
new file mode 100644
index 0000000..803a756
--- /dev/null
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationITAbstract.java
@@ -0,0 +1,376 @@
+// Copyright (C) 2020 The Android Open Source Project
+//
+// Licensed 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 com.googlesource.gerrit.plugins.replication.pull;
+
+import static com.google.common.truth.Truth.assertThat;
+import static com.google.gerrit.acceptance.GitUtil.fetch;
+import static com.google.gerrit.acceptance.GitUtil.pushOne;
+import static com.google.gerrit.acceptance.testsuite.project.TestProjectUpdate.allow;
+import static com.google.gerrit.server.group.SystemGroupBackend.REGISTERED_USERS;
+
+import com.google.gerrit.acceptance.PushOneCommit.Result;
+import com.google.gerrit.acceptance.UseLocalDisk;
+import com.google.gerrit.acceptance.config.GerritConfig;
+import com.google.gerrit.entities.Permission;
+import com.google.gerrit.entities.Project.NameKey;
+import com.google.gerrit.entities.RefNames;
+import com.google.gerrit.extensions.api.changes.NotifyHandling;
+import com.google.gerrit.extensions.api.projects.BranchInput;
+import com.google.gerrit.extensions.events.HeadUpdatedListener;
+import com.google.gerrit.extensions.events.ProjectDeletedListener;
+import com.google.gerrit.extensions.restapi.RestApiException;
+import com.googlesource.gerrit.plugins.replication.AutoReloadConfigDecorator;
+import java.io.IOException;
+import java.util.Collection;
+import java.util.Collections;
+import java.util.List;
+import java.util.Optional;
+import org.eclipse.jgit.internal.storage.dfs.InMemoryRepository;
+import org.eclipse.jgit.junit.TestRepository;
+import org.eclipse.jgit.lib.ObjectId;
+import org.eclipse.jgit.lib.Ref;
+import org.eclipse.jgit.lib.Repository;
+import org.eclipse.jgit.revwalk.RevCommit;
+import org.eclipse.jgit.transport.PushResult;
+import org.eclipse.jgit.transport.RemoteRefUpdate;
+import org.eclipse.jgit.transport.RemoteRefUpdate.Status;
+import org.junit.Test;
+
+/** Base class to run regular and async acceptance tests */
+public abstract class PullReplicationITAbstract extends PullReplicationSetupBase {
+
+  @Override
+  protected void setReplicationSource(
+      String remoteName, List<String> replicaSuffixes, Optional<String> project)
+      throws IOException {
+    List<String> fetchUrls =
+        buildReplicaURLs(replicaSuffixes, s -> gitPath.resolve("${name}" + s + ".git").toString());
+    config.setStringList("remote", remoteName, "url", fetchUrls);
+    config.setString("remote", remoteName, "apiUrl", adminRestSession.url());
+    config.setString("remote", remoteName, "fetch", "+refs/*:refs/*");
+    config.setInt("remote", remoteName, "timeout", 600);
+    config.setInt("remote", remoteName, "replicationDelay", TEST_REPLICATION_DELAY);
+    project.ifPresent(prj -> config.setString("remote", remoteName, "projects", prj));
+    config.setBoolean("gerrit", null, "autoReload", true);
+    config.save();
+  }
+
+  @Override
+  public void setUpTestPlugin() throws Exception {
+    setUpTestPlugin(false);
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  public void shouldReplicateNewChangeRef() throws Exception {
+    testRepo = cloneProject(createTestProject(project + TEST_REPLICATION_SUFFIX));
+
+    Result pushResult = createChange();
+    RevCommit sourceCommit = pushResult.getCommit();
+    String sourceRef = pushResult.getPatchSet().refName();
+
+    ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
+    FakeGitReferenceUpdatedEvent event =
+        new FakeGitReferenceUpdatedEvent(
+            project,
+            sourceRef,
+            ObjectId.zeroId().getName(),
+            sourceCommit.getId().getName(),
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
+
+    try (Repository repo = repoManager.openRepository(project)) {
+      waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
+
+      Ref targetBranchRef = getRef(repo, sourceRef);
+      assertThat(targetBranchRef).isNotNull();
+      assertThat(targetBranchRef.getObjectId()).isEqualTo(sourceCommit.getId());
+    }
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  public void shouldReplicateNewBranch() throws Exception {
+    String testProjectName = project + TEST_REPLICATION_SUFFIX;
+    createTestProject(testProjectName);
+
+    String newBranch = "refs/heads/mybranch";
+    String master = "refs/heads/master";
+    BranchInput input = new BranchInput();
+    input.revision = master;
+    gApi.projects().name(testProjectName).branch(newBranch).create(input);
+    String branchRevision = gApi.projects().name(testProjectName).branch(newBranch).get().revision;
+
+    ReplicationQueue pullReplicationQueue =
+        plugin.getSysInjector().getInstance(ReplicationQueue.class);
+    FakeGitReferenceUpdatedEvent event =
+        new FakeGitReferenceUpdatedEvent(
+            project,
+            newBranch,
+            ObjectId.zeroId().getName(),
+            branchRevision,
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
+
+    try (Repository repo = repoManager.openRepository(project);
+        Repository sourceRepo = repoManager.openRepository(project)) {
+      waitUntil(() -> checkedGetRef(repo, newBranch) != null);
+
+      Ref targetBranchRef = getRef(repo, newBranch);
+      assertThat(targetBranchRef).isNotNull();
+      assertThat(targetBranchRef.getObjectId().getName()).isEqualTo(branchRevision);
+    }
+  }
+
+  @Test
+  @UseLocalDisk
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  public void shouldReplicateForceUpdatedBranch() throws Exception {
+    boolean forcedPush = true;
+    String testProjectName = project + TEST_REPLICATION_SUFFIX;
+    NameKey testProjectNameKey = createTestProject(testProjectName);
+
+    String newBranch = "refs/heads/mybranch";
+    String master = "refs/heads/master";
+    BranchInput input = new BranchInput();
+    input.revision = master;
+    gApi.projects().name(testProjectName).branch(newBranch).create(input);
+
+    projectOperations
+        .project(testProjectNameKey)
+        .forUpdate()
+        .add(allow(Permission.PUSH).ref(newBranch).group(REGISTERED_USERS).force(true))
+        .update();
+
+    String branchRevision = gApi.projects().name(testProjectName).branch(newBranch).get().revision;
+
+    ReplicationQueue pullReplicationQueue =
+        plugin.getSysInjector().getInstance(ReplicationQueue.class);
+    FakeGitReferenceUpdatedEvent event =
+        new FakeGitReferenceUpdatedEvent(
+            project,
+            newBranch,
+            ObjectId.zeroId().getName(),
+            branchRevision,
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
+
+    try (Repository repo = repoManager.openRepository(project)) {
+      waitUntil(() -> checkedGetRef(repo, newBranch) != null);
+
+      Ref targetBranchRef = getRef(repo, newBranch);
+      assertThat(targetBranchRef).isNotNull();
+      assertThat(targetBranchRef.getObjectId().getName()).isEqualTo(branchRevision);
+    }
+
+    TestRepository<InMemoryRepository> testProject = cloneProject(testProjectNameKey);
+    fetch(testProject, RefNames.REFS_HEADS + "*:" + RefNames.REFS_HEADS + "*");
+    RevCommit amendedCommit = testProject.amendRef(newBranch).message("Amended commit").create();
+    PushResult pushResult =
+        pushOne(testProject, newBranch, newBranch, false, forcedPush, Collections.emptyList());
+    Collection<RemoteRefUpdate> pushedRefs = pushResult.getRemoteUpdates();
+    assertThat(pushedRefs).hasSize(1);
+    assertThat(pushedRefs.iterator().next().getStatus()).isEqualTo(Status.OK);
+
+    FakeGitReferenceUpdatedEvent forcedPushEvent =
+        new FakeGitReferenceUpdatedEvent(
+            project,
+            newBranch,
+            branchRevision,
+            amendedCommit.getId().getName(),
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(forcedPushEvent);
+
+    try (Repository repo = repoManager.openRepository(project);
+        Repository sourceRepo = repoManager.openRepository(project)) {
+      waitUntil(
+          () ->
+              checkedGetRef(repo, newBranch) != null
+                  && checkedGetRef(repo, newBranch)
+                      .getObjectId()
+                      .getName()
+                      .equals(amendedCommit.getId().getName()));
+    }
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  public void shouldReplicateNewChangeRefCGitClient() throws Exception {
+    AutoReloadConfigDecorator autoReloadConfigDecorator =
+        getInstance(AutoReloadConfigDecorator.class);
+
+    config.setBoolean("replication", null, "useCGitClient", true);
+    config.save();
+
+    autoReloadConfigDecorator.reload();
+
+    testRepo = cloneProject(createTestProject(project + TEST_REPLICATION_SUFFIX));
+
+    Result pushResult = createChange();
+    RevCommit sourceCommit = pushResult.getCommit();
+    String sourceRef = pushResult.getPatchSet().refName();
+
+    ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
+    FakeGitReferenceUpdatedEvent event =
+        new FakeGitReferenceUpdatedEvent(
+            project,
+            sourceRef,
+            ObjectId.zeroId().getName(),
+            sourceCommit.getId().getName(),
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
+
+    try (Repository repo = repoManager.openRepository(project)) {
+      waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
+
+      Ref targetBranchRef = getRef(repo, sourceRef);
+      assertThat(targetBranchRef).isNotNull();
+      assertThat(targetBranchRef.getObjectId()).isEqualTo(sourceCommit.getId());
+    }
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  public void shouldReplicateNewBranchCGitClient() throws Exception {
+    AutoReloadConfigDecorator autoReloadConfigDecorator =
+        getInstance(AutoReloadConfigDecorator.class);
+
+    config.setBoolean("replication", null, "useCGitClient", true);
+    config.save();
+
+    autoReloadConfigDecorator.reload();
+
+    String testProjectName = project + TEST_REPLICATION_SUFFIX;
+    createTestProject(testProjectName);
+
+    String newBranch = "refs/heads/mybranch";
+    String master = "refs/heads/master";
+    BranchInput input = new BranchInput();
+    input.revision = master;
+    gApi.projects().name(testProjectName).branch(newBranch).create(input);
+    String branchRevision = gApi.projects().name(testProjectName).branch(newBranch).get().revision;
+
+    ReplicationQueue pullReplicationQueue =
+        plugin.getSysInjector().getInstance(ReplicationQueue.class);
+    FakeGitReferenceUpdatedEvent event =
+        new FakeGitReferenceUpdatedEvent(
+            project,
+            newBranch,
+            ObjectId.zeroId().getName(),
+            branchRevision,
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
+
+    try (Repository repo = repoManager.openRepository(project);
+        Repository sourceRepo = repoManager.openRepository(project)) {
+      waitUntil(() -> checkedGetRef(repo, newBranch) != null);
+
+      Ref targetBranchRef = getRef(repo, newBranch);
+      assertThat(targetBranchRef).isNotNull();
+      assertThat(targetBranchRef.getObjectId().getName()).isEqualTo(branchRevision);
+    }
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  public void shouldReplicateProjectDeletion() throws Exception {
+    String projectToDelete = project.get();
+    setReplicationSource(TEST_REPLICATION_REMOTE, "", Optional.of(projectToDelete));
+    config.save();
+    AutoReloadConfigDecorator autoReloadConfigDecorator =
+        getInstance(AutoReloadConfigDecorator.class);
+    autoReloadConfigDecorator.reload();
+
+    ProjectDeletedListener.Event event =
+        new ProjectDeletedListener.Event() {
+          @Override
+          public String getProjectName() {
+            return projectToDelete;
+          }
+
+          @Override
+          public NotifyHandling getNotify() {
+            return NotifyHandling.NONE;
+          }
+        };
+    for (ProjectDeletedListener l : deletedListeners) {
+      l.onProjectDeleted(event);
+    }
+
+    waitUntil(() -> !repoManager.list().contains(project));
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  public void shouldReplicateHeadUpdate() throws Exception {
+    String testProjectName = project.get();
+    setReplicationSource(TEST_REPLICATION_REMOTE, "", Optional.of(testProjectName));
+    config.save();
+    AutoReloadConfigDecorator autoReloadConfigDecorator =
+        getInstance(AutoReloadConfigDecorator.class);
+    autoReloadConfigDecorator.reload();
+
+    String newBranch = "refs/heads/mybranch";
+    String master = "refs/heads/master";
+    BranchInput input = new BranchInput();
+    input.revision = master;
+    gApi.projects().name(testProjectName).branch(newBranch).create(input);
+    String branchRevision = gApi.projects().name(testProjectName).branch(newBranch).get().revision;
+
+    ReplicationQueue pullReplicationQueue =
+        plugin.getSysInjector().getInstance(ReplicationQueue.class);
+
+    HeadUpdatedListener.Event event = new FakeHeadUpdateEvent(master, newBranch, testProjectName);
+    pullReplicationQueue.onHeadUpdated(event);
+
+    waitUntil(
+        () -> {
+          try {
+            return gApi.projects().name(testProjectName).head().equals(newBranch);
+          } catch (RestApiException e) {
+            return false;
+          }
+        });
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = TEST_REPLICATION_REMOTE)
+  @GerritConfig(name = "container.replica", value = "true")
+  public void shouldReplicateNewChangeRefToReplica() throws Exception {
+    testRepo = cloneProject(createTestProject(project + TEST_REPLICATION_SUFFIX));
+
+    Result pushResult = createChange();
+    RevCommit sourceCommit = pushResult.getCommit();
+    String sourceRef = pushResult.getPatchSet().refName();
+
+    ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
+    FakeGitReferenceUpdatedEvent event =
+        new FakeGitReferenceUpdatedEvent(
+            project,
+            sourceRef,
+            ObjectId.zeroId().getName(),
+            sourceCommit.getId().getName(),
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
+
+    try (Repository repo = repoManager.openRepository(project)) {
+      waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
+
+      Ref targetBranchRef = getRef(repo, sourceRef);
+      assertThat(targetBranchRef).isNotNull();
+      assertThat(targetBranchRef.getObjectId()).isEqualTo(sourceCommit.getId());
+    }
+  }
+}
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationWithGitHttpTransportProtocolIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationWithGitHttpTransportProtocolIT.java
index 2d1f51f..e55e383 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationWithGitHttpTransportProtocolIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PullReplicationWithGitHttpTransportProtocolIT.java
@@ -21,7 +21,6 @@
 import com.google.gerrit.acceptance.TestPlugin;
 import com.google.gerrit.acceptance.UseLocalDisk;
 import com.google.gerrit.acceptance.config.GerritConfig;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener;
 import java.io.IOException;
 import java.util.List;
 import java.util.Optional;
@@ -29,7 +28,6 @@
 import org.eclipse.jgit.lib.Ref;
 import org.eclipse.jgit.lib.Repository;
 import org.eclipse.jgit.revwalk.RevCommit;
-import org.eclipse.jgit.transport.ReceiveCommand;
 import org.junit.Test;
 
 @SkipProjectClone
@@ -82,14 +80,14 @@
     String sourceRef = pushResult.getPatchSet().refName();
 
     ReplicationQueue pullReplicationQueue = getInstance(ReplicationQueue.class);
-    GitBatchRefUpdateListener.Event event =
+    FakeGitReferenceUpdatedEvent event =
         new FakeGitReferenceUpdatedEvent(
             project,
             sourceRef,
             ObjectId.zeroId().getName(),
             sourceCommit.getId().getName(),
-            ReceiveCommand.Type.CREATE);
-    pullReplicationQueue.onGitBatchRefUpdate(event);
+            TEST_REPLICATION_REMOTE);
+    pullReplicationQueue.onEvent(event);
 
     try (Repository repo = repoManager.openRepository(project)) {
       waitUntil(() -> checkedGetRef(repo, sourceRef) != null);
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueueTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueueTest.java
index e7de264..9b7e8c1 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueueTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueueTest.java
@@ -17,11 +17,11 @@
 import static com.google.common.truth.Truth.assertThat;
 import static java.nio.file.Files.createTempDirectory;
 import static org.mockito.ArgumentMatchers.anyBoolean;
+import static org.mockito.ArgumentMatchers.anyLong;
 import static org.mockito.Mockito.any;
 import static org.mockito.Mockito.anyInt;
 import static org.mockito.Mockito.anyString;
 import static org.mockito.Mockito.eq;
-import static org.mockito.Mockito.inOrder;
 import static org.mockito.Mockito.lenient;
 import static org.mockito.Mockito.never;
 import static org.mockito.Mockito.times;
@@ -29,34 +29,36 @@
 import static org.mockito.Mockito.verifyZeroInteractions;
 import static org.mockito.Mockito.when;
 
+import com.google.common.base.Suppliers;
 import com.google.common.collect.ImmutableList;
 import com.google.common.collect.Lists;
 import com.google.gerrit.entities.Project;
 import com.google.gerrit.extensions.api.changes.NotifyHandling;
 import com.google.gerrit.extensions.common.AccountInfo;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener;
-import com.google.gerrit.extensions.events.GitBatchRefUpdateListener.UpdatedRef;
 import com.google.gerrit.extensions.events.ProjectDeletedListener;
 import com.google.gerrit.extensions.registration.DynamicItem;
 import com.google.gerrit.metrics.DisabledMetricMaker;
 import com.google.gerrit.server.config.SitePaths;
+import com.google.gerrit.server.data.RefUpdateAttribute;
+import com.google.gerrit.server.events.Event;
 import com.google.gerrit.server.events.EventDispatcher;
+import com.google.gerrit.server.events.RefUpdatedEvent;
 import com.google.gerrit.server.git.WorkQueue;
 import com.google.inject.Provider;
 import com.googlesource.gerrit.plugins.replication.ReplicationConfig;
 import com.googlesource.gerrit.plugins.replication.ReplicationFileBasedConfig;
 import com.googlesource.gerrit.plugins.replication.pull.api.data.RevisionData;
+import com.googlesource.gerrit.plugins.replication.pull.api.exception.RefUpdateException;
 import com.googlesource.gerrit.plugins.replication.pull.client.FetchApiClient;
 import com.googlesource.gerrit.plugins.replication.pull.client.FetchRestApiClient;
 import com.googlesource.gerrit.plugins.replication.pull.client.HttpResult;
+import com.googlesource.gerrit.plugins.replication.pull.filter.ApplyObjectsRefsFilter;
 import com.googlesource.gerrit.plugins.replication.pull.filter.ExcludedRefsFilter;
 import java.io.IOException;
 import java.nio.file.Path;
 import java.util.Arrays;
 import java.util.List;
 import java.util.Optional;
-import java.util.Set;
-import java.util.stream.Collectors;
 import org.apache.http.client.ClientProtocolException;
 import org.eclipse.jgit.errors.LargeObjectException;
 import org.eclipse.jgit.lib.ObjectId;
@@ -67,13 +69,15 @@
 import org.junit.runner.RunWith;
 import org.mockito.ArgumentCaptor;
 import org.mockito.Captor;
-import org.mockito.InOrder;
 import org.mockito.Mock;
 import org.mockito.junit.MockitoJUnitRunner;
 
 @RunWith(MockitoJUnitRunner.class)
 public class ReplicationQueueTest {
   private static int CONNECTION_TIMEOUT = 1000000;
+  private static final String LOCAL_INSTANCE_ID = "local instance id";
+  private static final String FOREIGN_INSTANCE_ID = "any other instance id";
+  private static final String TEST_REF_NAME = "refs/meta/heads/anyref";
 
   @Mock private WorkQueue wq;
   @Mock private Source source;
@@ -91,6 +95,7 @@
   @Mock RevisionData revisionDataWithParents;
   List<ObjectId> revisionDataParentObjectIds;
   @Mock HttpResult httpResult;
+  @Mock ApplyObjectsRefsFilter applyObjectsRefsFilter;
   ApplyObjectMetrics applyObjectMetrics;
   FetchReplicationMetrics fetchMetrics;
 
@@ -135,10 +140,12 @@
 
     when(fetchClientFactory.create(any())).thenReturn(fetchRestApiClient);
     lenient()
-        .when(fetchRestApiClient.callSendObject(any(), anyString(), anyBoolean(), any(), any()))
+        .when(
+            fetchRestApiClient.callSendObject(
+                any(), anyString(), anyLong(), anyBoolean(), any(), any()))
         .thenReturn(httpResult);
     lenient()
-        .when(fetchRestApiClient.callSendObjects(any(), anyString(), any(), any()))
+        .when(fetchRestApiClient.callSendObjects(any(), anyString(), anyLong(), any(), any()))
         .thenReturn(httpResult);
     when(fetchRestApiClient.callFetch(any(), anyString(), any())).thenReturn(fetchHttpResult);
     when(fetchRestApiClient.initProject(any(), any())).thenReturn(successfulHttpResult);
@@ -146,6 +153,7 @@
     when(httpResult.isSuccessful()).thenReturn(true);
     when(fetchHttpResult.isSuccessful()).thenReturn(true);
     when(httpResult.isProjectMissing(any())).thenReturn(false);
+    when(applyObjectsRefsFilter.match(any())).thenReturn(false);
 
     applyObjectMetrics = new ApplyObjectMetrics("pull-replication", new DisabledMetricMaker());
     fetchMetrics = new FetchReplicationMetrics("pull-replication", new DisabledMetricMaker());
@@ -160,102 +168,89 @@
             refsFilter,
             () -> revReader,
             applyObjectMetrics,
-            fetchMetrics);
+            fetchMetrics,
+            LOCAL_INSTANCE_ID,
+            applyObjectsRefsFilter);
   }
 
   @Test
   public void shouldCallSendObjectWhenMetaRef() throws ClientProtocolException, IOException {
-    TestEvent event = new TestEvent("refs/changes/01/1/meta");
+    Event event = new TestEvent("refs/changes/01/1/meta");
     objectUnderTest.start();
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
-    verify(fetchRestApiClient).callSendObjects(any(), anyString(), any(), any());
+    verify(fetchRestApiClient).callSendObjects(any(), anyString(), anyLong(), any(), any());
+  }
+
+  @Test
+  public void shouldIgnoreEventWhenIsNotLocalInstanceId()
+      throws ClientProtocolException, IOException {
+    Event event = new TestEvent();
+    event.instanceId = FOREIGN_INSTANCE_ID;
+    objectUnderTest.start();
+    objectUnderTest.onEvent(event);
+
+    verify(fetchRestApiClient, never())
+        .callSendObjects(any(), anyString(), anyLong(), any(), any());
   }
 
   @Test
   public void shouldCallInitProjectWhenProjectIsMissing() throws IOException {
-    TestEvent event = new TestEvent("refs/changes/01/1/meta");
+    Event event = new TestEvent("refs/changes/01/1/meta");
     when(httpResult.isSuccessful()).thenReturn(false);
     when(httpResult.isProjectMissing(any())).thenReturn(true);
     when(source.isCreateMissingRepositories()).thenReturn(true);
 
     objectUnderTest.start();
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
     verify(fetchRestApiClient).initProject(any(), any());
   }
 
   @Test
   public void shouldNotCallInitProjectWhenReplicateNewRepositoriesNotSet() throws IOException {
-    TestEvent event = new TestEvent("refs/changes/01/1/meta");
+    Event event = new TestEvent("refs/changes/01/1/meta");
     when(httpResult.isSuccessful()).thenReturn(false);
     when(httpResult.isProjectMissing(any())).thenReturn(true);
     when(source.isCreateMissingRepositories()).thenReturn(false);
 
     objectUnderTest.start();
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
     verify(fetchRestApiClient, never()).initProject(any(), any());
   }
 
   @Test
   public void shouldCallSendObjectWhenPatchSetRef() throws ClientProtocolException, IOException {
-    TestEvent event = new TestEvent("refs/changes/01/1/1");
+    Event event = new TestEvent("refs/changes/01/1/1");
     objectUnderTest.start();
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
-    verify(fetchRestApiClient).callSendObjects(any(), anyString(), any(), any());
-  }
-
-  @Test
-  public void shouldCallSendObjectReorderingRefsHavingMetaAtTheEnd()
-      throws ClientProtocolException, IOException {
-    sendRefUpdatedEvents("refs/changes/01/1/meta", "refs/changes/01/1/1");
-    verifySendObjectOrdering("refs/changes/01/1/1", "refs/changes/01/1/meta");
-  }
-
-  @Test
-  public void shouldCallSendObjectKeepingMetaAtTheEnd()
-      throws ClientProtocolException, IOException {
-    sendRefUpdatedEvents("refs/changes/01/1/1", "refs/changes/01/1/meta");
-    verifySendObjectOrdering("refs/changes/01/1/1", "refs/changes/01/1/meta");
-  }
-
-  private void sendRefUpdatedEvents(String firstRef, String secondRef) {
-    objectUnderTest.start();
-    objectUnderTest.onGitBatchRefUpdate(new TestEvent(firstRef, secondRef));
-  }
-
-  private void verifySendObjectOrdering(String firstRef, String secondRef)
-      throws ClientProtocolException, IOException {
-    InOrder inOrder = inOrder(fetchRestApiClient);
-
-    inOrder.verify(fetchRestApiClient).callSendObjects(any(), eq(firstRef), any(), any());
-    inOrder.verify(fetchRestApiClient).callSendObjects(any(), eq(secondRef), any(), any());
+    verify(fetchRestApiClient).callSendObjects(any(), anyString(), anyLong(), any(), any());
   }
 
   @Test
   public void shouldFallbackToCallFetchWhenIOException()
-      throws ClientProtocolException, IOException, LargeObjectException {
-    TestEvent event = new TestEvent("refs/changes/01/1/meta");
+      throws ClientProtocolException, IOException, LargeObjectException, RefUpdateException {
+    Event event = new TestEvent("refs/changes/01/1/meta");
     objectUnderTest.start();
 
     when(revReader.read(any(), any(), anyString(), anyInt())).thenThrow(IOException.class);
 
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
     verify(fetchRestApiClient).callFetch(any(), anyString(), any());
   }
 
   @Test
   public void shouldFallbackToCallFetchWhenLargeRef()
-      throws ClientProtocolException, IOException, LargeObjectException {
-    TestEvent event = new TestEvent("refs/changes/01/1/1");
+      throws ClientProtocolException, IOException, LargeObjectException, RefUpdateException {
+    Event event = new TestEvent("refs/changes/01/1/1");
     objectUnderTest.start();
 
     when(revReader.read(any(), any(), anyString(), anyInt())).thenReturn(Optional.empty());
 
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
     verify(fetchRestApiClient).callFetch(any(), anyString(), any());
   }
@@ -263,15 +258,15 @@
   @Test
   public void shouldFallbackToCallFetchWhenParentObjectIsMissing()
       throws ClientProtocolException, IOException {
-    TestEvent event = new TestEvent("refs/changes/01/1/1");
+    Event event = new TestEvent("refs/changes/01/1/1");
     objectUnderTest.start();
 
     when(httpResult.isSuccessful()).thenReturn(false);
     when(httpResult.isParentObjectMissing()).thenReturn(true);
-    when(fetchRestApiClient.callSendObjects(any(), anyString(), any(), any()))
+    when(fetchRestApiClient.callSendObjects(any(), anyString(), anyLong(), any(), any()))
         .thenReturn(httpResult);
 
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
     verify(fetchRestApiClient).callFetch(any(), anyString(), any());
   }
@@ -279,18 +274,46 @@
   @Test
   public void shouldFallbackToApplyAllParentObjectsWhenParentObjectIsMissingOnMetaRef()
       throws ClientProtocolException, IOException {
-    TestEvent event = new TestEvent("refs/changes/01/1/meta");
+    Event event = new TestEvent("refs/changes/01/1/meta");
     objectUnderTest.start();
 
     when(httpResult.isSuccessful()).thenReturn(false, true);
     when(httpResult.isParentObjectMissing()).thenReturn(true, false);
-    when(fetchRestApiClient.callSendObjects(any(), anyString(), any(), any()))
+    when(fetchRestApiClient.callSendObjects(any(), anyString(), anyLong(), any(), any()))
         .thenReturn(httpResult);
 
-    objectUnderTest.onGitBatchRefUpdate(event);
+    objectUnderTest.onEvent(event);
 
     verify(fetchRestApiClient, times(2))
-        .callSendObjects(any(), anyString(), revisionsDataCaptor.capture(), any());
+        .callSendObjects(any(), anyString(), anyLong(), revisionsDataCaptor.capture(), any());
+    List<List<RevisionData>> revisionsDataValues = revisionsDataCaptor.getAllValues();
+    assertThat(revisionsDataValues).hasSize(2);
+
+    List<RevisionData> firstRevisionsValues = revisionsDataValues.get(0);
+    assertThat(firstRevisionsValues).hasSize(1);
+    assertThat(firstRevisionsValues).contains(revisionData);
+
+    List<RevisionData> secondRevisionsValues = revisionsDataValues.get(1);
+    assertThat(secondRevisionsValues).hasSize(1 + revisionDataParentObjectIds.size());
+  }
+
+  @Test
+  public void shouldFallbackToApplyAllParentObjectsWhenParentObjectIsMissingOnAllowedRefs()
+      throws ClientProtocolException, IOException {
+    String refName = "refs/tags/test-tag";
+    Event event = new TestEvent(refName);
+    objectUnderTest.start();
+
+    when(httpResult.isSuccessful()).thenReturn(false, true);
+    when(httpResult.isParentObjectMissing()).thenReturn(true, false);
+    when(fetchRestApiClient.callSendObjects(any(), anyString(), anyLong(), any(), any()))
+        .thenReturn(httpResult);
+    when(applyObjectsRefsFilter.match(refName)).thenReturn(true);
+
+    objectUnderTest.onEvent(event);
+
+    verify(fetchRestApiClient, times(2))
+        .callSendObjects(any(), anyString(), anyLong(), revisionsDataCaptor.capture(), any());
     List<List<RevisionData>> revisionsDataValues = revisionsDataCaptor.getAllValues();
     assertThat(revisionsDataValues).hasSize(2);
 
@@ -321,17 +344,19 @@
             refsFilter,
             () -> revReader,
             applyObjectMetrics,
-            fetchMetrics);
-    TestEvent event = new TestEvent("refs/multi-site/version");
-    objectUnderTest.onGitBatchRefUpdate(event);
+            fetchMetrics,
+            LOCAL_INSTANCE_ID,
+            applyObjectsRefsFilter);
+    Event event = new TestEvent("refs/multi-site/version");
+    objectUnderTest.onEvent(event);
 
     verifyZeroInteractions(wq, rd, dis, sl, fetchClientFactory, accountInfo);
   }
 
   @Test
   public void shouldSkipEventWhenStarredChangesRef() {
-    TestEvent event = new TestEvent("refs/starred-changes/41/2941/1000000");
-    objectUnderTest.onGitBatchRefUpdate(event);
+    Event event = new TestEvent("refs/starred-changes/41/2941/1000000");
+    objectUnderTest.onEvent(event);
 
     verifyZeroInteractions(wq, rd, dis, sl, fetchClientFactory, accountInfo);
   }
@@ -396,83 +421,27 @@
     return createTempDirectory(prefix);
   }
 
-  private static class TestEvent implements GitBatchRefUpdateListener.Event {
-    private String refName;
-    private String projectName;
-    private List<UpdatedRef> refs;
+  private class TestEvent extends RefUpdatedEvent {
 
-    public TestEvent(String... refNames) {
+    public TestEvent() {
+      this(TEST_REF_NAME);
+    }
+
+    public TestEvent(String refName) {
       this(
+          refName,
           "defaultProject",
-          Arrays.stream(refNames)
-              .map(refName -> updateRef(refName, ObjectId.zeroId()))
-              .collect(Collectors.toUnmodifiableList()));
+          ObjectId.fromString("3c1ddc050d7906adb0e29bc3bc46af8749b2f63b"));
     }
 
-    public TestEvent(String projectName, List<UpdatedRef> refs) {
-      this.projectName = projectName;
-      this.refs = refs;
-    }
-
-    @Override
-    public String getProjectName() {
-      return projectName;
-    }
-
-    @Override
-    public NotifyHandling getNotify() {
-      return null;
-    }
-
-    @Override
-    public AccountInfo getUpdater() {
-      return null;
-    }
-
-    @Override
-    public Set<UpdatedRef> getUpdatedRefs() {
-      return refs.stream().collect(Collectors.toSet());
-    }
-
-    private static final GitBatchRefUpdateListener.UpdatedRef updateRef(
-        String refName, ObjectId refObjectId) {
-      return new GitBatchRefUpdateListener.UpdatedRef() {
-
-        @Override
-        public String getRefName() {
-          return refName;
-        }
-
-        @Override
-        public String getOldObjectId() {
-          return ObjectId.zeroId().getName();
-        }
-
-        @Override
-        public String getNewObjectId() {
-          return refObjectId.getName();
-        }
-
-        @Override
-        public boolean isCreate() {
-          return false;
-        }
-
-        @Override
-        public boolean isDelete() {
-          return false;
-        }
-
-        @Override
-        public boolean isNonFastForward() {
-          return false;
-        }
-      };
-    }
-
-    @Override
-    public Set<String> getRefNames() {
-      return Set.of(refName);
+    public TestEvent(String refName, String projectName, ObjectId newObjectId) {
+      RefUpdateAttribute upd = new RefUpdateAttribute();
+      upd.newRev = newObjectId.getName();
+      upd.oldRev = ObjectId.zeroId().getName();
+      upd.project = projectName;
+      upd.refName = refName;
+      this.refUpdate = Suppliers.ofInstance(upd);
+      this.instanceId = LOCAL_INSTANCE_ID;
     }
   }
 
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionIT.java
index 2ab5caf..453a6b0 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionIT.java
@@ -27,11 +27,12 @@
 public class ApplyObjectActionIT extends ActionITBase {
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldAcceptPayloadWithAsyncField() throws Exception {
     String payloadWithAsyncFieldTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}, \"async\":true}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}, \"async\":true}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -48,11 +49,12 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldAcceptPayloadWithoutAsyncField() throws Exception {
     String payloadWithoutAsyncFieldTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -70,12 +72,13 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldAcceptPayloadWhenNodeIsAReplica() throws Exception {
     String payloadWithoutAsyncFieldTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -93,12 +96,13 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldAcceptPayloadWhenNodeIsAReplicaAndProjectNameContainsSlash() throws Exception {
     String payloadWithoutAsyncFieldTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
     NameKey projectName = Project.nameKey("test/repo");
     String refName = createRef(projectName);
     Optional<RevisionData> revisionDataOption = createRevisionData(projectName, refName);
@@ -119,12 +123,13 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnForbiddenWhenNodeIsAReplicaAndUSerIsAnonymous() throws Exception {
     String payloadWithoutAsyncFieldTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -140,9 +145,10 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnBadRequestCodeWhenMandatoryFieldLabelIsMissing() throws Exception {
     String payloadWithoutLabelFieldTemplate =
-        "{\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}, \"async\":true}";
+        "{\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}, \"async\":true}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -160,11 +166,12 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnBadRequestCodeWhenPayloadIsNotAProperJSON() throws Exception {
     String wrongPayloadTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}, \"async\":true,}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}, \"async\":true,}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -181,6 +188,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldAcceptPayloadWhenNodeIsAReplicaWithBearerToken() throws Exception {
@@ -188,7 +196,7 @@
     String payloadWithoutAsyncFieldTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -206,6 +214,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "false")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldAcceptPayloadWhenNodeIsAPrimaryWithBearerToken() throws Exception {
@@ -213,7 +222,7 @@
     String payloadWithoutAsyncFieldTemplate =
         "{\"label\":\""
             + TEST_REPLICATION_REMOTE
-            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
+            + "\",\"ref_name\":\"%s\",\"revision_data\":{\"commit_object\":{\"sha1\":\"%s\",\"type\":1,\"content\":\"%s\"},\"tree_object\":{\"type\":2,\"content\":\"%s\"},\"blobs\":[]}}";
 
     String refName = createRef();
     Optional<RevisionData> revisionDataOption = createRevisionData(refName);
@@ -236,6 +245,7 @@
         String.format(
             wrongPayloadTemplate,
             refName,
+            revisionData.getCommitObject().getSha1(),
             encode(revisionData.getCommitObject().getContent()),
             encode(revisionData.getTreeObject().getContent()));
     return sendObjectPayload;
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionTest.java
index 814ba76..6ef5b2a 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectActionTest.java
@@ -17,6 +17,7 @@
 import static com.google.common.truth.Truth.assertThat;
 import static org.apache.http.HttpStatus.SC_CREATED;
 import static org.mockito.ArgumentMatchers.any;
+import static org.mockito.ArgumentMatchers.anyLong;
 import static org.mockito.ArgumentMatchers.anyString;
 import static org.mockito.Mockito.doThrow;
 import static org.mockito.Mockito.when;
@@ -48,6 +49,9 @@
 
 @RunWith(MockitoJUnitRunner.class)
 public class ApplyObjectActionTest {
+
+  private static final long DUMMY_EVENT_TIMESTAMP = 1684875939;
+
   ApplyObjectAction applyObjectAction;
   String label = "instance-2-label";
   String url = "file:///gerrit-host/instance-1/git/${name}.git";
@@ -55,7 +59,6 @@
   String refMetaName = "refs/meta/version";
   String location = "http://gerrit-host/a/config/server/tasks/08d173e9";
   int taskId = 1234;
-
   private String sampleCommitObjectId = "9f8d52853089a3cf00c02ff7bd0817bd4353a95a";
   private String sampleTreeObjectId = "4b825dc642cb6eb9a060e54bf8d69288fbee4904";
   private String sampleBlobObjectId = "b5d7bcf1d1c5b0f0726d10a16c8315f06f900bfb";
@@ -95,7 +98,8 @@
 
   @Test
   public void shouldReturnCreatedResponseCode() throws RestApiException {
-    RevisionInput inputParams = new RevisionInput(label, refName, createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput(label, refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
 
     Response<?> response = applyObjectAction.apply(projectResource, inputParams);
 
@@ -109,6 +113,7 @@
         new RevisionInput(
             label,
             refMetaName,
+            DUMMY_EVENT_TIMESTAMP,
             createSampleRevisionDataBlob(
                 new RevisionObjectData(sampleBlobObjectId, Constants.OBJ_BLOB, blobData)));
 
@@ -120,7 +125,8 @@
   @SuppressWarnings("cast")
   @Test
   public void shouldReturnSourceUrlAndrefNameAsAResponseBody() throws Exception {
-    RevisionInput inputParams = new RevisionInput(label, refName, createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput(label, refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
     Response<?> response = applyObjectAction.apply(projectResource, inputParams);
 
     assertThat((RevisionInput) response.value()).isEqualTo(inputParams);
@@ -128,28 +134,32 @@
 
   @Test(expected = BadRequestException.class)
   public void shouldThrowBadRequestExceptionWhenMissingLabel() throws Exception {
-    RevisionInput inputParams = new RevisionInput(null, refName, createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput(null, refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
 
     applyObjectAction.apply(projectResource, inputParams);
   }
 
   @Test(expected = BadRequestException.class)
   public void shouldThrowBadRequestExceptionWhenEmptyLabel() throws Exception {
-    RevisionInput inputParams = new RevisionInput("", refName, createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput("", refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
 
     applyObjectAction.apply(projectResource, inputParams);
   }
 
   @Test(expected = BadRequestException.class)
   public void shouldThrowBadRequestExceptionWhenMissingRefName() throws Exception {
-    RevisionInput inputParams = new RevisionInput(label, null, createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput(label, null, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
 
     applyObjectAction.apply(projectResource, inputParams);
   }
 
   @Test(expected = BadRequestException.class)
   public void shouldThrowBadRequestExceptionWhenEmptyRefName() throws Exception {
-    RevisionInput inputParams = new RevisionInput(label, "", createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput(label, "", DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
 
     applyObjectAction.apply(projectResource, inputParams);
   }
@@ -161,7 +171,8 @@
     RevisionObjectData treeData =
         new RevisionObjectData(sampleTreeObjectId, Constants.OBJ_TREE, new byte[] {});
     RevisionInput inputParams =
-        new RevisionInput(label, refName, createSampleRevisionData(commitData, treeData));
+        new RevisionInput(
+            label, refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData(commitData, treeData));
 
     applyObjectAction.apply(projectResource, inputParams);
   }
@@ -172,7 +183,8 @@
         new RevisionObjectData(
             sampleCommitObjectId, Constants.OBJ_COMMIT, sampleCommitContent.getBytes());
     RevisionInput inputParams =
-        new RevisionInput(label, refName, createSampleRevisionData(commitData, null));
+        new RevisionInput(
+            label, refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData(commitData, null));
 
     applyObjectAction.apply(projectResource, inputParams);
   }
@@ -180,7 +192,8 @@
   @Test(expected = AuthException.class)
   public void shouldThrowAuthExceptionWhenCallFetchActionCapabilityNotAssigned()
       throws RestApiException {
-    RevisionInput inputParams = new RevisionInput(label, refName, createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput(label, refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
 
     when(preConditions.canCallFetchApi()).thenReturn(false);
 
@@ -190,13 +203,14 @@
   @Test(expected = ResourceConflictException.class)
   public void shouldThrowResourceConflictExceptionWhenMissingParentObject()
       throws RestApiException, IOException, RefUpdateException, MissingParentObjectException {
-    RevisionInput inputParams = new RevisionInput(label, refName, createSampleRevisionData());
+    RevisionInput inputParams =
+        new RevisionInput(label, refName, DUMMY_EVENT_TIMESTAMP, createSampleRevisionData());
 
     doThrow(
             new MissingParentObjectException(
                 Project.nameKey("test_projects"), refName, ObjectId.zeroId()))
         .when(applyObjectCommand)
-        .applyObject(any(), anyString(), any(), anyString());
+        .applyObject(any(), anyString(), any(), anyString(), anyLong());
 
     applyObjectAction.apply(projectResource, inputParams);
   }
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommandTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommandTest.java
index 4c188df..7f5a67c 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommandTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ApplyObjectCommandTest.java
@@ -20,6 +20,8 @@
 import static org.mockito.Mockito.verify;
 import static org.mockito.Mockito.when;
 
+import com.google.common.cache.Cache;
+import com.google.common.cache.CacheBuilder;
 import com.google.common.collect.Lists;
 import com.google.gerrit.entities.Project;
 import com.google.gerrit.entities.Project.NameKey;
@@ -29,6 +31,7 @@
 import com.google.gerrit.server.events.EventDispatcher;
 import com.google.gerrit.server.permissions.PermissionBackendException;
 import com.googlesource.gerrit.plugins.replication.pull.ApplyObjectMetrics;
+import com.googlesource.gerrit.plugins.replication.pull.ApplyObjectsCacheKey;
 import com.googlesource.gerrit.plugins.replication.pull.FetchRefReplicatedEvent;
 import com.googlesource.gerrit.plugins.replication.pull.PullReplicationStateLogger;
 import com.googlesource.gerrit.plugins.replication.pull.Source;
@@ -61,9 +64,13 @@
   private static final NameKey TEST_PROJECT_NAME = Project.nameKey("test-project");
   private static final String TEST_REMOTE_NAME = "test-remote-name";
   private static URIish TEST_REMOTE_URI;
+  private static final long TEST_EVENT_TIMESTAMP = 1L;
 
   private String sampleCommitObjectId = "9f8d52853089a3cf00c02ff7bd0817bd4353a95a";
   private String sampleTreeObjectId = "4b825dc642cb6eb9a060e54bf8d69288fbee4904";
+  private String sampleBlobObjectId = "b5d7bcf1d1c5b0f0726d10a16c8315f06f900bfb";
+  private String sampleCommitObjectId2 = "9f8d52853089a3cf00c02ff7bd0817bd4353a95b";
+  private String sampleTreeObjectId2 = "4b825dc642cb6eb9a060e54bf8d69288fbee4905";
 
   @Mock private PullReplicationStateLogger fetchStateLog;
   @Mock private ApplyObject applyObject;
@@ -74,11 +81,13 @@
   @Mock private SourcesCollection sourceCollection;
   @Mock private Source source;
   @Captor ArgumentCaptor<Event> eventCaptor;
+  private Cache<ApplyObjectsCacheKey, Long> cache;
 
   private ApplyObjectCommand objectUnderTest;
 
   @Before
   public void setup() throws MissingParentObjectException, IOException, URISyntaxException {
+    cache = CacheBuilder.newBuilder().build();
     RefUpdateState state = new RefUpdateState(TEST_REMOTE_NAME, RefUpdate.Result.NEW);
     TEST_REMOTE_URI = new URIish("git://some.remote.uri");
     when(eventDispatcherDataItem.get()).thenReturn(eventDispatcher);
@@ -90,15 +99,21 @@
 
     objectUnderTest =
         new ApplyObjectCommand(
-            fetchStateLog, applyObject, metrics, eventDispatcherDataItem, sourceCollection);
+            fetchStateLog, applyObject, metrics, eventDispatcherDataItem, sourceCollection, cache);
   }
 
   @Test
   public void shouldSendEventWhenApplyObject()
       throws PermissionBackendException, IOException, RefUpdateException,
           MissingParentObjectException {
+    RevisionData sampleRevisionData =
+        createSampleRevisionData(sampleCommitObjectId, sampleTreeObjectId);
     objectUnderTest.applyObject(
-        TEST_PROJECT_NAME, TEST_REF_NAME, createSampleRevisionData(), TEST_SOURCE_LABEL);
+        TEST_PROJECT_NAME,
+        TEST_REF_NAME,
+        sampleRevisionData,
+        TEST_SOURCE_LABEL,
+        TEST_EVENT_TIMESTAMP);
 
     verify(eventDispatcher).postEvent(eventCaptor.capture());
     Event sentEvent = eventCaptor.getValue();
@@ -109,11 +124,64 @@
     assertThat(fetchEvent.targetUri).isEqualTo(TEST_REMOTE_URI.toASCIIString());
   }
 
-  private RevisionData createSampleRevisionData() {
+  @Test
+  public void shouldInsertIntoApplyObjectsCacheWhenApplyObjectIsSuccessful()
+      throws IOException, RefUpdateException, MissingParentObjectException {
+    RevisionData sampleRevisionData =
+        createSampleRevisionData(sampleCommitObjectId, sampleTreeObjectId);
+    RevisionData sampleRevisionData2 =
+        createSampleRevisionData(sampleCommitObjectId2, sampleTreeObjectId2);
+    objectUnderTest.applyObjects(
+        TEST_PROJECT_NAME,
+        TEST_REF_NAME,
+        new RevisionData[] {sampleRevisionData, sampleRevisionData2},
+        TEST_SOURCE_LABEL,
+        TEST_EVENT_TIMESTAMP);
+
+    assertThat(
+            cache.getIfPresent(
+                ApplyObjectsCacheKey.create(
+                    sampleRevisionData.getCommitObject().getSha1(),
+                    TEST_REF_NAME,
+                    TEST_PROJECT_NAME.get())))
+        .isEqualTo(TEST_EVENT_TIMESTAMP);
+    assertThat(
+            cache.getIfPresent(
+                ApplyObjectsCacheKey.create(
+                    sampleRevisionData2.getCommitObject().getSha1(),
+                    TEST_REF_NAME,
+                    TEST_PROJECT_NAME.get())))
+        .isEqualTo(TEST_EVENT_TIMESTAMP);
+  }
+
+  @Test(expected = RefUpdateException.class)
+  public void shouldNotInsertIntoApplyObjectsCacheWhenApplyObjectIsFailure()
+      throws IOException, RefUpdateException, MissingParentObjectException {
+    RevisionData sampleRevisionData =
+        createSampleRevisionData(sampleCommitObjectId, sampleTreeObjectId);
+    RefUpdateState failureState = new RefUpdateState(TEST_REMOTE_NAME, RefUpdate.Result.IO_FAILURE);
+    when(applyObject.apply(any(), any(), any())).thenReturn(failureState);
+    objectUnderTest.applyObject(
+        TEST_PROJECT_NAME,
+        TEST_REF_NAME,
+        sampleRevisionData,
+        TEST_SOURCE_LABEL,
+        TEST_EVENT_TIMESTAMP);
+
+    assertThat(
+            cache.getIfPresent(
+                ApplyObjectsCacheKey.create(
+                    sampleRevisionData.getCommitObject().getSha1(),
+                    TEST_REF_NAME,
+                    TEST_PROJECT_NAME.get())))
+        .isNull();
+  }
+
+  private RevisionData createSampleRevisionData(String commitObjectId, String treeObjectId) {
     RevisionObjectData commitData =
-        new RevisionObjectData(sampleCommitObjectId, Constants.OBJ_COMMIT, new byte[] {});
+        new RevisionObjectData(commitObjectId, Constants.OBJ_COMMIT, new byte[] {});
     RevisionObjectData treeData =
-        new RevisionObjectData(sampleTreeObjectId, Constants.OBJ_TREE, new byte[] {});
+        new RevisionObjectData(treeObjectId, Constants.OBJ_TREE, new byte[] {});
     return new RevisionData(Collections.emptyList(), commitData, treeData, Lists.newArrayList());
   }
 }
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchActionIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchActionIT.java
index cc01c2a..0a69c3d 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchActionIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchActionIT.java
@@ -23,6 +23,7 @@
 public class FetchActionIT extends ActionITBase {
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldFetchRefWhenNodeIsAReplica() throws Exception {
     String refName = createRef();
@@ -41,6 +42,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldFetchRefWhenNodeIsAReplicaAndProjectNameContainsSlash() throws Exception {
     NameKey projectName = Project.nameKey("test/repo");
@@ -63,6 +65,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnForbiddenWhenNodeIsAReplicaAndUSerIsAnonymous() throws Exception {
     String refName = createRef();
@@ -79,6 +82,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldFetchRefWhenNodeIsAReplicaWithBearerToken() throws Exception {
@@ -99,6 +103,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "false")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldFetchRefWhenNodeIsAPrimaryWithBearerToken() throws Exception {
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectDeletionActionIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectDeletionActionIT.java
index de8a942..87a8048 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectDeletionActionIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectDeletionActionIT.java
@@ -32,6 +32,7 @@
   @Inject private ProjectOperations projectOperations;
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnUnauthorizedForUserWithoutPermissions() throws Exception {
     httpClientFactory
         .create(source)
@@ -40,6 +41,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldDeleteRepositoryWhenUserHasProjectDeletionCapabilities() throws Exception {
     String testProjectName = project.get();
     url = getURLWithAuthenticationPrefix(testProjectName);
@@ -63,6 +65,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnOKWhenProjectIsDeleted() throws Exception {
     String testProjectName = project.get();
     url = getURLWithAuthenticationPrefix(testProjectName);
@@ -75,6 +78,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnBadRequestWhenDeletingAnInvalidProjectName() throws Exception {
     url = getURLWithAuthenticationPrefix(INVALID_TEST_PROJECT_NAME);
 
@@ -86,6 +90,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnForbiddenForUserWithoutPermissionsOnReplica() throws Exception {
     httpClientFactory
@@ -94,6 +99,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnOKWhenProjectIsDeletedOnReplica() throws Exception {
     String testProjectName = project.get();
@@ -107,6 +113,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldDeleteRepositoryWhenUserHasProjectDeletionCapabilitiesAndNodeIsAReplica()
       throws Exception {
@@ -130,6 +137,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnBadRequestWhenDeletingAnInvalidProjectNameWhenNodeIsAReplica()
       throws Exception {
@@ -143,6 +151,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldReturnOKWhenProjectIsDeletedOnReplicaWithBearerToken() throws Exception {
@@ -157,6 +166,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "false")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldReturnOKWhenProjectIsDeletedOnPrimaryWithBearerToken() throws Exception {
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectInitializationActionIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectInitializationActionIT.java
index 47037d6..c543969 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectInitializationActionIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ProjectInitializationActionIT.java
@@ -31,9 +31,11 @@
 import org.junit.Test;
 
 public class ProjectInitializationActionIT extends ActionITBase {
+  public static final String INVALID_TEST_PROJECT_NAME = "\0";
   @Inject private ProjectOperations projectOperations;
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnUnauthorizedForUserWithoutPermissions() throws Exception {
     httpClientFactory
         .create(source)
@@ -43,6 +45,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnBadRequestIfContentNotSet() throws Exception {
     httpClientFactory
         .create(source)
@@ -52,6 +55,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldCreateRepository() throws Exception {
     String newProjectName = "new/newProjectForPrimary";
     url = getURLWithAuthenticationPrefix(newProjectName);
@@ -71,6 +75,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldCreateRepositoryWhenUserHasProjectCreationCapabilities() throws Exception {
     String newProjectName = "new/newProjectForUserWithCapabilities";
     url = getURLWithAuthenticationPrefix(newProjectName);
@@ -93,6 +98,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnForbiddenIfUserNotAuthorized() throws Exception {
     httpClientFactory
         .create(source)
@@ -102,6 +108,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldCreateRepositoryWhenNodeIsAReplica() throws Exception {
     String newProjectName = "new/newProjectForReplica";
@@ -114,6 +121,8 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnForbiddenIfUserNotAuthorizedAndNodeIsAReplica() throws Exception {
     httpClientFactory
@@ -124,6 +133,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldCreateRepositoryWhenUserHasProjectCreationCapabilitiesAndNodeIsAReplica()
       throws Exception {
@@ -148,6 +158,20 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
+  @GerritConfig(name = "container.replica", value = "true")
+  public void shouldReturnBadRequestIfProjectNameIsInvalidAndCannotBeCreatedWhenNodeIsAReplica()
+      throws Exception {
+    url = getURLWithAuthenticationPrefix(INVALID_TEST_PROJECT_NAME);
+    httpClientFactory
+        .create(source)
+        .execute(
+            withBasicAuthenticationAsAdmin(createPutRequestWithHeaders()),
+            assertHttpResponseCode(HttpServletResponse.SC_BAD_REQUEST));
+  }
+
+  @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnBadRequestIfContentNotSetWhenNodeIsAReplica() throws Exception {
     httpClientFactory
@@ -158,6 +182,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnForbiddenForUserWithoutPermissionsWhenNodeIsAReplica() throws Exception {
     httpClientFactory
@@ -168,6 +193,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldCreateRepositoryWhenNodeIsAReplicaWithBearerToken() throws Exception {
@@ -181,6 +207,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "false")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   public void shouldCreateRepositoryWhenNodeIsAPrimaryWithBearerToken() throws Exception {
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/UpdateHeadActionIT.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/UpdateHeadActionIT.java
index 7c725b3..f6e631f 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/UpdateHeadActionIT.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/UpdateHeadActionIT.java
@@ -36,6 +36,7 @@
   @Inject private ProjectOperations projectOperations;
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnUnauthorizedForUserWithoutPermissions() throws Exception {
     httpClientFactory
         .create(source)
@@ -45,6 +46,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnBadRequestWhenInputIsEmpty() throws Exception {
     httpClientFactory
         .create(source)
@@ -54,6 +56,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnOKWhenHeadIsUpdated() throws Exception {
     String testProjectName = project.get();
     String newBranch = "refs/heads/mybranch";
@@ -71,6 +74,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnBadRequestWhenInputIsEmptyInReplica() throws Exception {
     httpClientFactory
@@ -81,6 +85,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnOKWhenHeadIsUpdatedInReplica() throws Exception {
     String testProjectName = project.get();
@@ -99,6 +104,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnForbiddenWhenMissingPermissions() throws Exception {
     httpClientFactory
         .create(source)
@@ -108,6 +114,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   public void shouldReturnOKWhenRegisteredUserHasPermissions() throws Exception {
     String testProjectName = project.get();
     String newBranch = "refs/heads/mybranch";
@@ -132,6 +139,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   public void shouldReturnForbiddenWhenMissingPermissionsInReplica() throws Exception {
     httpClientFactory
@@ -142,6 +150,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "true")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   @Ignore("Waiting for resolving: Issue 16332: Not able to update the HEAD from internal user")
@@ -164,6 +173,7 @@
   }
 
   @Test
+  @GerritConfig(name = "gerrit.instanceId", value = "testInstanceId")
   @GerritConfig(name = "container.replica", value = "false")
   @GerritConfig(name = "auth.bearerToken", value = "some-bearer-token")
   @Ignore("Waiting for resolving: Issue 16332: Not able to update the HEAD from internal user")
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClientBase.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClientBase.java
index a2389d7..cdb238e 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClientBase.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/client/FetchRestApiClientBase.java
@@ -74,6 +74,7 @@
   String pluginName = "pull-replication";
   String instanceId = "Replication";
   String refName = RefNames.REFS_HEADS + "master";
+  long eventCreatedOn = 1684875939;
 
   String expectedPayload =
       "{\"label\":\"Replication\", \"ref_name\": \"" + refName + "\", \"async\":false}";
@@ -87,7 +88,9 @@
   String blobObjectId = "bb383f5249c68a4cc8c82bdd1228b4a8883ff6e8";
 
   String expectedSendObjectPayload =
-      "{\"label\":\"Replication\",\"ref_name\":\"refs/heads/master\",\"revision_data\":{\"commit_object\":{\"sha1\":\""
+      "{\"label\":\"Replication\",\"ref_name\":\"refs/heads/master\",\"event_created_on\":"
+          + eventCreatedOn
+          + ",\"revision_data\":{\"commit_object\":{\"sha1\":\""
           + commitObjectId
           + "\",\"type\":1,\"content\":\"dHJlZSA3NzgxNGQyMTZhNmNhYjJkZGI5ZjI4NzdmYmJkMGZlYmRjMGZhNjA4CnBhcmVudCA5ODNmZjFhM2NmNzQ3MjVhNTNhNWRlYzhkMGMwNjEyMjEyOGY1YThkCmF1dGhvciBHZXJyaXQgVXNlciAxMDAwMDAwIDwxMDAwMDAwQDY5ZWMzOGYwLTM1MGUtNGQ5Yy05NmQ0LWJjOTU2ZjJmYWFhYz4gMTYxMDU3ODY0OCArMDEwMApjb21taXR0ZXIgR2Vycml0IENvZGUgUmV2aWV3IDxyb290QG1hY3plY2gtWFBTLTE1PiAxNjEwNTc4NjQ4ICswMTAwCgpVcGRhdGUgcGF0Y2ggc2V0IDEKClBhdGNoIFNldCAxOgoKKDEgY29tbWVudCkKClBhdGNoLXNldDogMQo\\u003d\"},\"tree_object\":{\"sha1\":\""
           + treeObjectId
@@ -265,6 +268,7 @@
     objectUnderTest.callSendObject(
         Project.nameKey("test_repo"),
         refName,
+        eventCreatedOn,
         IS_REF_UPDATE,
         createSampleRevisionData(),
         new URIish(api));
@@ -287,6 +291,7 @@
     objectUnderTest.callSendObject(
         Project.nameKey("test_repo"),
         refName,
+        eventCreatedOn,
         IS_REF_UPDATE,
         createSampleRevisionData(),
         new URIish(api));
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListenerTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListenerTest.java
index 92314a4..8dc0359 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListenerTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/event/StreamEventListenerTest.java
@@ -21,6 +21,8 @@
 import static org.mockito.Mockito.verify;
 import static org.mockito.Mockito.when;
 
+import com.google.common.cache.Cache;
+import com.google.common.cache.CacheBuilder;
 import com.google.common.collect.Lists;
 import com.google.gerrit.entities.Project;
 import com.google.gerrit.entities.RefNames;
@@ -32,6 +34,7 @@
 import com.google.gerrit.server.events.RefUpdatedEvent;
 import com.google.gerrit.server.git.WorkQueue;
 import com.google.gerrit.server.permissions.PermissionBackendException;
+import com.googlesource.gerrit.plugins.replication.pull.ApplyObjectsCacheKey;
 import com.googlesource.gerrit.plugins.replication.pull.FetchOne;
 import com.googlesource.gerrit.plugins.replication.pull.Source;
 import com.googlesource.gerrit.plugins.replication.pull.SourcesCollection;
@@ -60,6 +63,7 @@
   private static final String INSTANCE_ID = "node_instance_id";
   private static final String NEW_REV = "0000000000000000000000000000000000000001";
   private static final String REMOTE_INSTANCE_ID = "remote_node_instance_id";
+  private static final long TEST_EVENT_TIMESTAMP = 1684879097024L;
 
   @Mock private ProjectInitializationAction projectInitializationAction;
   @Mock private WorkQueue workQueue;
@@ -72,11 +76,13 @@
   @Mock private SourcesCollection sources;
   @Mock private Source source;
   @Mock private ExcludedRefsFilter refsFilter;
+  private Cache<ApplyObjectsCacheKey, Long> cache;
 
   private StreamEventListener objectUnderTest;
 
   @Before
   public void setup() {
+    cache = CacheBuilder.newBuilder().build();
     when(workQueue.getDefaultQueue()).thenReturn(executor);
     when(fetchJobFactory.create(eq(Project.nameKey(TEST_PROJECT)), any(), any()))
         .thenReturn(fetchJob);
@@ -95,7 +101,8 @@
             fetchJobFactory,
             () -> metrics,
             sources,
-            refsFilter);
+            refsFilter,
+            cache);
   }
 
   @Test
@@ -257,4 +264,58 @@
 
     verify(executor).submit(any(FetchJob.class));
   }
+
+  @Test
+  public void shouldSkipEventWhenFoundInApplyObjectsCacheWithTheSameTimestamp() {
+    sendRefUpdateEventWithTimestamp(TEST_EVENT_TIMESTAMP, TEST_EVENT_TIMESTAMP);
+    verify(executor, never()).submit(any(Runnable.class));
+  }
+
+  @Test
+  public void shouldSkipEventWhenFoundInApplyObjectsCacheWithOlderTimestamp() {
+    sendRefUpdateEventWithTimestamp(TEST_EVENT_TIMESTAMP - 1, TEST_EVENT_TIMESTAMP);
+    verify(executor, never()).submit(any(Runnable.class));
+  }
+
+  @Test
+  public void shouldProcessEventWhenFoundInApplyObjectsCacheWithNewerTimestamp() {
+    sendRefUpdateEventWithTimestamp(TEST_EVENT_TIMESTAMP + 1, TEST_EVENT_TIMESTAMP);
+    verify(executor).submit(any(Runnable.class));
+  }
+
+  private void sendRefUpdateEventWithTimestamp(long eventTimestamp, long cachedTimestamp) {
+    RefUpdatedEvent event = new RefUpdatedEvent();
+    RefUpdateAttribute refUpdate = new RefUpdateAttribute();
+    refUpdate.refName = TEST_REF_NAME;
+    refUpdate.project = TEST_PROJECT;
+    refUpdate.oldRev = ObjectId.zeroId().getName();
+    refUpdate.newRev = NEW_REV;
+    event.eventCreatedOn = eventTimestamp;
+
+    cache.put(
+        ApplyObjectsCacheKey.create(refUpdate.newRev, refUpdate.refName, refUpdate.project),
+        cachedTimestamp);
+
+    event.instanceId = REMOTE_INSTANCE_ID;
+    event.refUpdate = () -> refUpdate;
+
+    objectUnderTest.onEvent(event);
+  }
+
+  @Test
+  public void shouldScheduleAllRefsFetchWhenNotFoundInApplyObjectsCache() {
+    RefUpdatedEvent event = new RefUpdatedEvent();
+    RefUpdateAttribute refUpdate = new RefUpdateAttribute();
+    refUpdate.refName = TEST_REF_NAME;
+    refUpdate.project = TEST_PROJECT;
+    refUpdate.oldRev = ObjectId.zeroId().getName();
+    refUpdate.newRev = NEW_REV;
+
+    event.instanceId = REMOTE_INSTANCE_ID;
+    event.refUpdate = () -> refUpdate;
+
+    objectUnderTest.onEvent(event);
+
+    verify(executor).submit(any(FetchJob.class));
+  }
 }