Merge branch 'stable-3.5' into stable-3.6

* stable-3.5:
  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: Ibb4e88e652dcec616b5ba65ac85fd713800a5272
diff --git a/example-setup/broker/Dockerfile b/example-setup/broker/Dockerfile
index 08eaba9..b79470c 100644
--- a/example-setup/broker/Dockerfile
+++ b/example-setup/broker/Dockerfile
@@ -1,4 +1,4 @@
-FROM gerritcodereview/gerrit:3.5.5-almalinux8
+FROM gerritcodereview/gerrit:3.6.3-almalinux8
 
 USER root
 
@@ -12,7 +12,7 @@
 # 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 libevents-broker.jar /var/gerrit/lib/libevents-broker.jar
+COPY --chown=gerrit:gerrit events-broker.jar /var/gerrit/lib/events-broker.jar
 
 COPY --chown=gerrit:gerrit entrypoint.sh /tmp/
 COPY --chown=gerrit:gerrit configs/replication.config.template /var/gerrit/etc/
diff --git a/example-setup/http/Dockerfile b/example-setup/http/Dockerfile
index e9f8239..77fed72 100644
--- a/example-setup/http/Dockerfile
+++ b/example-setup/http/Dockerfile
@@ -1,4 +1,4 @@
-FROM gerritcodereview/gerrit:3.5.5-almalinux8
+FROM gerritcodereview/gerrit:3.6.3-almalinux8
 
 USER root
 
diff --git a/external_plugin_deps.bzl b/external_plugin_deps.bzl
index 5d2ba13..3bf728d 100644
--- a/external_plugin_deps.bzl
+++ b/external_plugin_deps.bzl
@@ -3,6 +3,6 @@
 def external_plugin_deps():
     maven_jar(
         name = "events-broker",
-        artifact = "com.gerritforge:events-broker:3.5.0.1",
-        sha1 = "af192a8bceaf7ff54d19356f9bfe1f1e83634b40",
+        artifact = "com.gerritforge:events-broker:3.6.0-rc3",
+        sha1 = "cb398afa4f76367be5c62b99a7ffce74ae1d3d8b",
     )
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 fc77503..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
@@ -170,12 +170,7 @@
             event.getRefName(),
             event.refUpdate.get().oldRev,
             event.refUpdate.get().newRev);
-        fire(
-            event.refUpdate.get().project,
-            ObjectId.fromString(event.refUpdate.get().newRev),
-            event.getRefName(),
-            event.eventCreatedOn,
-            ZEROS_OBJECTID.equals(event.refUpdate.get().newRev));
+        fire(ReferenceUpdatedEvent.from(event));
       }
     }
   }
@@ -204,40 +199,40 @@
     return !refsFilter.match(refName);
   }
 
-  private void fire(
-      String projectName,
-      ObjectId objectId,
-      String refName,
-      long eventCreatedOn,
-      boolean isDelete) {
+  private void fire(ReferenceUpdatedEvent event) {
     ReplicationState state = new ReplicationState(new GitUpdateProcessing(dispatcher.get()));
-    fire(Project.nameKey(projectName), objectId, refName, eventCreatedOn, isDelete, state);
+    fire(event, state);
     state.markAllFetchTasksScheduled();
   }
 
-  private void fire(
-      NameKey project,
-      ObjectId objectId,
-      String refName,
-      long eventCreatedOn,
-      boolean isDelete,
-      ReplicationState state) {
+  private void fire(ReferenceUpdatedEvent event, ReplicationState state) {
     if (!running) {
       stateLog.warn(
           "Replication plugin did not finish startup before event, event replication is postponed",
           state);
-      beforeStartupEventsQueue.add(
-          ReferenceUpdatedEvent.create(project.get(), refName, objectId, eventCreatedOn, isDelete));
+      beforeStartupEventsQueue.add(event);
       return;
     }
     ForkJoinPool fetchCallsPool = null;
     try {
-      fetchCallsPool = new ForkJoinPool(sources.get().getAll().size());
+      List<Source> allSources = sources.get().getAll();
+      int numSources = allSources.size();
+      if (numSources == 0) {
+        repLog.debug("No replication sources configured -> skipping fetch");
+        return;
+      }
+      fetchCallsPool = new ForkJoinPool(numSources);
 
       final Consumer<Source> callFunction =
-          callFunction(project, objectId, refName, eventCreatedOn, isDelete, state);
+          callFunction(
+              Project.nameKey(event.projectName()),
+              event.objectId(),
+              event.refName(),
+              event.eventCreatedOn(),
+              event.isDelete(),
+              state);
       fetchCallsPool
-          .submit(() -> sources.get().getAll().parallelStream().forEach(callFunction))
+          .submit(() -> allSources.parallelStream().forEach(callFunction))
           .get(fetchCallsTimeout, TimeUnit.MILLISECONDS);
     } catch (InterruptedException | ExecutionException | TimeoutException e) {
       stateLog.error(
@@ -526,12 +521,7 @@
       String eventKey = String.format("%s:%s", event.projectName(), event.refName());
       if (!eventsReplayed.contains(eventKey)) {
         repLog.info("Firing pending task {}", event);
-        fire(
-            event.projectName(),
-            event.objectId(),
-            event.refName(),
-            event.eventCreatedOn(),
-            event.isDelete());
+        fire(event);
         eventsReplayed.add(eventKey);
       }
     }
@@ -561,6 +551,15 @@
           projectName, refName, objectId, eventCreatedOn, isDelete);
     }
 
+    static ReferenceUpdatedEvent from(RefUpdatedEvent event) {
+      return ReferenceUpdatedEvent.create(
+          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();
 
     public abstract String refName();
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/SourceConfigParser.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/SourceConfigParser.java
index a8799c2..d7ae063 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/SourceConfigParser.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/SourceConfigParser.java
@@ -17,6 +17,8 @@
 import com.google.common.collect.ImmutableList;
 import com.google.common.collect.Lists;
 import com.google.common.flogger.FluentLogger;
+import com.google.gerrit.server.config.GerritIsReplica;
+import com.google.inject.Inject;
 import com.googlesource.gerrit.plugins.replication.ConfigParser;
 import com.googlesource.gerrit.plugins.replication.RemoteConfiguration;
 import java.net.URISyntaxException;
@@ -32,6 +34,13 @@
 
   private static final FluentLogger logger = FluentLogger.forEnclosingClass();
 
+  private boolean isReplica;
+
+  @Inject
+  SourceConfigParser(@GerritIsReplica Boolean isReplica) {
+    this.isReplica = isReplica;
+  }
+
   /* (non-Javadoc)
    * @see com.googlesource.gerrit.plugins.replication.ConfigParser#parseRemotes(org.eclipse.jgit.lib.Config)
    */
@@ -45,7 +54,7 @@
 
     ImmutableList.Builder<RemoteConfiguration> sourceConfigs = ImmutableList.builder();
     for (RemoteConfig c : allFetchRemotes(config)) {
-      if (c.getURIs().isEmpty()) {
+      if (isReplica && c.getURIs().isEmpty()) {
         continue;
       }
 
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommand.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommand.java
index f897012..e49c8b6 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommand.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommand.java
@@ -35,7 +35,6 @@
 import com.googlesource.gerrit.plugins.replication.pull.ReplicationState;
 import com.googlesource.gerrit.plugins.replication.pull.Source;
 import com.googlesource.gerrit.plugins.replication.pull.SourcesCollection;
-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.Optional;
@@ -49,7 +48,6 @@
   private static final FluentLogger logger = FluentLogger.forEnclosingClass();
 
   private final PullReplicationStateLogger fetchStateLog;
-  private final ApplyObject applyObject;
   private final DynamicItem<EventDispatcher> eventDispatcher;
   private final ProjectCache projectCache;
   private final SourcesCollection sourcesCollection;
@@ -61,13 +59,11 @@
       PullReplicationStateLogger fetchStateLog,
       ProjectCache projectCache,
       SourcesCollection sourcesCollection,
-      ApplyObject applyObject,
       PermissionBackend permissionBackend,
       DynamicItem<EventDispatcher> eventDispatcher,
       LocalGitRepositoryManagerProvider gitManagerProvider) {
     this.fetchStateLog = fetchStateLog;
     this.projectCache = projectCache;
-    this.applyObject = applyObject;
     this.eventDispatcher = eventDispatcher;
     this.sourcesCollection = sourcesCollection;
     this.permissionBackend = permissionBackend;
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommand.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommand.java
index dd06875..991ef07 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommand.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommand.java
@@ -91,7 +91,12 @@
     try {
       state.markAllFetchTasksScheduled();
       Future<?> future = source.get().schedule(name, refName, state, fetchType, apiRequestMetrics);
-      future.get(source.get().getTimeout(), TimeUnit.SECONDS);
+      int timeout = source.get().getTimeout();
+      if (timeout == 0) {
+        future.get();
+      } else {
+        future.get(timeout, TimeUnit.SECONDS);
+      }
     } catch (ExecutionException
         | IllegalStateException
         | TimeoutException
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/CGitFetchValidator.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/CGitFetchValidator.java
index 9a10898..434c0f0 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/CGitFetchValidator.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/CGitFetchValidator.java
@@ -35,7 +35,7 @@
 
   @Override
   public Void visit(AssistedInjectBinding<? extends FetchFactory> binding) {
-    TypeLiteral<CGitFetch> nativeGitFetchType = new TypeLiteral<CGitFetch>() {};
+    TypeLiteral<CGitFetch> nativeGitFetchType = new TypeLiteral<>() {};
     for (AssistedMethod method : binding.getAssistedMethods()) {
       if (method.getImplementationType().equals(nativeGitFetchType)) {
         String[] command = new String[] {"git", "--version"};
diff --git a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/PermanentTransportException.java b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/PermanentTransportException.java
index acb68cf..0fa89b5 100644
--- a/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/PermanentTransportException.java
+++ b/src/main/java/com/googlesource/gerrit/plugins/replication/pull/fetch/PermanentTransportException.java
@@ -14,7 +14,7 @@
 
 package com.googlesource.gerrit.plugins.replication.pull.fetch;
 
-import com.jcraft.jsch.JSchException;
+import org.apache.sshd.common.SshException;
 import org.eclipse.jgit.errors.TransportException;
 
 public class PermanentTransportException extends TransportException {
@@ -26,7 +26,8 @@
 
   public static TransportException wrapIfPermanentTransportException(TransportException e) {
     Throwable cause = e.getCause();
-    if (cause instanceof JSchException && cause.getMessage().startsWith("UnknownHostKey:")) {
+    if (cause instanceof SshException
+        && cause.getMessage().startsWith("Failed (UnsupportedCredentialItem) to execute:")) {
       return new PermanentTransportException("Terminal fetch failure", e);
     }
 
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FetchGitUpdateProcessingTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FetchGitUpdateProcessingTest.java
index 68044b4..5a26054 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FetchGitUpdateProcessingTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/FetchGitUpdateProcessingTest.java
@@ -47,8 +47,7 @@
   }
 
   @Test
-  public void headRefReplicatedInGitUpdateProcessing()
-      throws URISyntaxException, PermissionBackendException {
+  public void headRefReplicatedInGitUpdateProcessing() throws PermissionBackendException {
     FetchRefReplicatedEvent expectedEvent =
         new FetchRefReplicatedEvent(
             "someProject",
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PermanentFailureExceptionTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PermanentFailureExceptionTest.java
index 09a465c..fcb0702 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PermanentFailureExceptionTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/PermanentFailureExceptionTest.java
@@ -18,7 +18,7 @@
 
 import com.googlesource.gerrit.plugins.replication.pull.fetch.InexistentRefTransportException;
 import com.googlesource.gerrit.plugins.replication.pull.fetch.PermanentTransportException;
-import com.jcraft.jsch.JSchException;
+import org.apache.sshd.common.SshException;
 import org.eclipse.jgit.errors.TransportException;
 import org.junit.Test;
 
@@ -29,7 +29,9 @@
     assertThat(
             PermanentTransportException.wrapIfPermanentTransportException(
                 new TransportException(
-                    "SSH error", new JSchException("UnknownHostKey: some.place"))))
+                    "SSH error",
+                    new SshException(
+                        "Failed (UnsupportedCredentialItem) to execute: some.commands"))))
         .isInstanceOf(PermanentTransportException.class);
   }
 
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 6ee3c43..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
@@ -1,4 +1,4 @@
-// Copyright (C) 2022 The Android Open Source Project
+// 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.
@@ -14,9 +14,41 @@
 
 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.SkipProjectClone;
 import com.google.gerrit.acceptance.TestPlugin;
 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;
 
 @SkipProjectClone
 @UseLocalDisk
@@ -24,4 +56,327 @@
     name = "pull-replication",
     sysModule = "com.googlesource.gerrit.plugins.replication.pull.PullReplicationModule",
     httpModule = "com.googlesource.gerrit.plugins.replication.pull.api.HttpModule")
-public class PullReplicationIT extends PullReplicationITAbstract {}
+public class PullReplicationIT 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);
+
+    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/ReplicationQueueTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/ReplicationQueueTest.java
index d65322f..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
@@ -362,7 +362,7 @@
   }
 
   @Test
-  public void shouldCallDeleteWhenReplicateProjectDeletionsTrue() throws IOException {
+  public void shouldCallDeleteWhenReplicateProjectDeletionsTrue() {
     when(source.wouldDeleteProject(any())).thenReturn(true);
 
     String projectName = "testProject";
@@ -378,7 +378,7 @@
   }
 
   @Test
-  public void shouldNotCallDeleteWhenProjectNotToDelete() throws IOException {
+  public void shouldNotCallDeleteWhenProjectNotToDelete() {
     when(source.wouldDeleteProject(any())).thenReturn(false);
 
     FakeProjectDeletedEvent event = new FakeProjectDeletedEvent("testProject");
@@ -390,7 +390,7 @@
   }
 
   @Test
-  public void shouldScheduleUpdateHeadWhenWouldFetchProject() throws IOException {
+  public void shouldScheduleUpdateHeadWhenWouldFetchProject() {
     when(source.wouldFetchProject(any())).thenReturn(true);
 
     String projectName = "aProject";
@@ -406,7 +406,7 @@
   }
 
   @Test
-  public void shouldNotScheduleUpdateHeadWhenNotWouldFetchProject() throws IOException {
+  public void shouldNotScheduleUpdateHeadWhenNotWouldFetchProject() {
     when(source.wouldFetchProject(any())).thenReturn(false);
 
     String projectName = "aProject";
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ActionITBase.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ActionITBase.java
index 20c1fea..e638653 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ActionITBase.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/ActionITBase.java
@@ -163,7 +163,7 @@
   }
 
   public ResponseHandler<Object> assertHttpResponseCode(int responseCode) {
-    return new ResponseHandler<Object>() {
+    return new ResponseHandler<>() {
 
       @Override
       public Object handleResponse(HttpResponse response)
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommandTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommandTest.java
index f1f9e44..fc1b02c 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommandTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/DeleteRefCommandTest.java
@@ -102,7 +102,6 @@
             fetchStateLog,
             projectCache,
             sourceCollection,
-            applyObject,
             permissionBackend,
             eventDispatcherDataItem,
             new LocalGitRepositoryManagerProvider(gitManager));
diff --git a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommandTest.java b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommandTest.java
index c093719..de8036e 100644
--- a/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommandTest.java
+++ b/src/test/java/com/googlesource/gerrit/plugins/replication/pull/api/FetchCommandTest.java
@@ -125,7 +125,7 @@
   @Test
   public void shouldUpdateStateWhenInterruptedException()
       throws InterruptedException, ExecutionException, TimeoutException {
-    when(future.get(anyLong(), eq(TimeUnit.SECONDS))).thenThrow(new InterruptedException());
+    when(future.get()).thenThrow(new InterruptedException());
     when(source.schedule(projectName, REF_NAME_TO_FETCH, state, SYNC, Optional.empty()))
         .thenReturn(future);
 
@@ -140,8 +140,7 @@
   @Test
   public void shouldUpdateStateWhenExecutionException()
       throws InterruptedException, ExecutionException, TimeoutException {
-    when(future.get(anyLong(), eq(TimeUnit.SECONDS)))
-        .thenThrow(new ExecutionException(new Exception()));
+    when(future.get()).thenThrow(new ExecutionException(new Exception()));
     when(source.schedule(projectName, REF_NAME_TO_FETCH, state, SYNC, Optional.empty()))
         .thenReturn(future);
 
@@ -159,6 +158,7 @@
     when(future.get(anyLong(), eq(TimeUnit.SECONDS))).thenThrow(new TimeoutException());
     when(source.schedule(projectName, REF_NAME_TO_FETCH, state, SYNC, Optional.empty()))
         .thenReturn(future);
+    when(source.getTimeout()).thenReturn(1);
 
     TimeoutException e =
         assertThrows(