diff --git a/server/src/internalClusterTest/java/org/elasticsearch/snapshots/SnapshotShutdownIT.java b/server/src/internalClusterTest/java/org/elasticsearch/snapshots/SnapshotShutdownIT.java index 1af8ac72b36bf..bbb392d97d8ae 100644 --- a/server/src/internalClusterTest/java/org/elasticsearch/snapshots/SnapshotShutdownIT.java +++ b/server/src/internalClusterTest/java/org/elasticsearch/snapshots/SnapshotShutdownIT.java @@ -494,7 +494,6 @@ public void testSnapshotShutdownProgressTracker() throws Exception { final var indexName = randomIdentifier(); createIndexWithContent(indexName, indexSettings(numShards, 0).put(REQUIRE_NODE_NAME_SETTING, nodeForRemoval).build()); indexAllShardsToAnEqualOrGreaterMinimumSize(indexName, ByteSizeValue.of(2, ByteSizeUnit.KB).getBytes()); - logger.info("---> nodeForRemovalId: " + nodeForRemovalId + ", numShards: " + numShards); // Start the snapshot with blocking in place on the data node not to allow shard snapshots to finish yet. final var clusterService = internalCluster().getCurrentMasterNodeInstance(ClusterService.class); @@ -504,21 +503,7 @@ public void testSnapshotShutdownProgressTracker() throws Exception { waitForBlock(otherNode, repoName); - // Block on the master when a shard snapshot request comes in, until we can verify that the Tracker saw the outgoing request. - final CountDownLatch snapshotStatusUpdateLatch = new CountDownLatch(1); - final var masterTransportService = MockTransportService.getInstance(internalCluster().getMasterName()); - masterTransportService.addRequestHandlingBehavior( - TransportUpdateSnapshotStatusAction.NAME, - (handler, request, channel, task) -> masterTransportService.getThreadPool().generic().execute(() -> { - safeAwait(snapshotStatusUpdateLatch); - try { - handler.messageReceived(request, channel, task); - } catch (Exception e) { - fail(e); - } - }) - ); - + logger.info("---> nodeForRemovalId: " + nodeForRemovalId + ", numShards: " + numShards); mockLog.addExpectation( new MockLog.SeenEventExpectation( "SnapshotShutdownProgressTracker start log message", @@ -566,6 +551,21 @@ public void testSnapshotShutdownProgressTracker() throws Exception { mockLog.awaitAllExpectationsMatched(); resetMockLog(); + // Block on the master when a shard snapshot request comes in, until we can verify that the Tracker saw the outgoing request. + final CountDownLatch snapshotStatusUpdateLatch = new CountDownLatch(1); + final var masterTransportService = MockTransportService.getInstance(internalCluster().getMasterName()); + masterTransportService.addRequestHandlingBehavior( + TransportUpdateSnapshotStatusAction.NAME, + (handler, request, channel, task) -> masterTransportService.getThreadPool().generic().execute(() -> { + safeAwait(snapshotStatusUpdateLatch); + try { + handler.messageReceived(request, channel, task); + } catch (Exception e) { + fail(e); + } + }) + ); + mockLog.addExpectation( new MockLog.SeenEventExpectation( "SnapshotShutdownProgressTracker shard snapshot has paused log message",