mirror of https://github.com/apache/kafka.git
KAFKA-16785 Migrate TopicBasedRemoteLogMetadataManagerRestartTest to new test infra (#16170)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
This commit is contained in:
parent
9eb05fc729
commit
b05f82d444
|
@ -53,12 +53,6 @@ public class TopicBasedRemoteLogMetadataManagerHarness extends IntegrationTestHa
|
||||||
initializeRemoteLogMetadataManager(topicIdPartitions, startConsumerThread, RemoteLogMetadataTopicPartitioner::new, remotePartitionMetadataStoreSupplier);
|
initializeRemoteLogMetadataManager(topicIdPartitions, startConsumerThread, RemoteLogMetadataTopicPartitioner::new, remotePartitionMetadataStoreSupplier);
|
||||||
}
|
}
|
||||||
|
|
||||||
public void initializeRemoteLogMetadataManager(Set<TopicIdPartition> topicIdPartitions,
|
|
||||||
boolean startConsumerThread,
|
|
||||||
Function<Integer, RemoteLogMetadataTopicPartitioner> remoteLogMetadataTopicPartitioner) {
|
|
||||||
initializeRemoteLogMetadataManager(topicIdPartitions, startConsumerThread, remoteLogMetadataTopicPartitioner, RemotePartitionMetadataStore::new);
|
|
||||||
}
|
|
||||||
|
|
||||||
public void initializeRemoteLogMetadataManager(Set<TopicIdPartition> topicIdPartitions,
|
public void initializeRemoteLogMetadataManager(Set<TopicIdPartition> topicIdPartitions,
|
||||||
boolean startConsumerThread,
|
boolean startConsumerThread,
|
||||||
Function<Integer, RemoteLogMetadataTopicPartitioner> remoteLogMetadataTopicPartitioner,
|
Function<Integer, RemoteLogMetadataTopicPartitioner> remoteLogMetadataTopicPartitioner,
|
||||||
|
@ -70,6 +64,7 @@ public class TopicBasedRemoteLogMetadataManagerHarness extends IntegrationTestHa
|
||||||
.startConsumerThread(startConsumerThread)
|
.startConsumerThread(startConsumerThread)
|
||||||
.remoteLogMetadataTopicPartitioner(remoteLogMetadataTopicPartitioner)
|
.remoteLogMetadataTopicPartitioner(remoteLogMetadataTopicPartitioner)
|
||||||
.remotePartitionMetadataStore(remotePartitionMetadataStoreSupplier)
|
.remotePartitionMetadataStore(remotePartitionMetadataStoreSupplier)
|
||||||
|
.overrideRemoteLogMetadataManagerProps(overrideRemoteLogMetadataManagerProps())
|
||||||
.build();
|
.build();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
@ -16,6 +16,11 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.kafka.server.log.remote.metadata.storage;
|
package org.apache.kafka.server.log.remote.metadata.storage;
|
||||||
|
|
||||||
|
import kafka.test.ClusterInstance;
|
||||||
|
import kafka.test.annotation.ClusterTest;
|
||||||
|
import kafka.test.junit.ClusterTestExtensions;
|
||||||
|
import org.apache.kafka.clients.admin.Admin;
|
||||||
|
import org.apache.kafka.clients.admin.NewTopic;
|
||||||
import org.apache.kafka.common.TopicIdPartition;
|
import org.apache.kafka.common.TopicIdPartition;
|
||||||
import org.apache.kafka.common.TopicPartition;
|
import org.apache.kafka.common.TopicPartition;
|
||||||
import org.apache.kafka.common.Uuid;
|
import org.apache.kafka.common.Uuid;
|
||||||
|
@ -24,139 +29,97 @@ import org.apache.kafka.common.utils.Time;
|
||||||
import org.apache.kafka.server.log.remote.storage.RemoteLogSegmentId;
|
import org.apache.kafka.server.log.remote.storage.RemoteLogSegmentId;
|
||||||
import org.apache.kafka.server.log.remote.storage.RemoteLogSegmentMetadata;
|
import org.apache.kafka.server.log.remote.storage.RemoteLogSegmentMetadata;
|
||||||
import org.apache.kafka.test.TestUtils;
|
import org.apache.kafka.test.TestUtils;
|
||||||
import org.junit.jupiter.api.AfterEach;
|
|
||||||
import org.junit.jupiter.api.Assertions;
|
import org.junit.jupiter.api.Assertions;
|
||||||
import org.junit.jupiter.api.BeforeEach;
|
import org.junit.jupiter.api.Tag;
|
||||||
import org.junit.jupiter.api.Test;
|
import org.junit.jupiter.api.extension.ExtendWith;
|
||||||
import scala.collection.JavaConverters;
|
|
||||||
import scala.collection.Seq;
|
|
||||||
|
|
||||||
import java.io.IOException;
|
|
||||||
import java.util.ArrayList;
|
|
||||||
import java.util.Arrays;
|
import java.util.Arrays;
|
||||||
import java.util.Collections;
|
import java.util.Collections;
|
||||||
import java.util.HashMap;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Map;
|
|
||||||
|
|
||||||
import static org.apache.kafka.server.log.remote.metadata.storage.TopicBasedRemoteLogMetadataManagerConfig.LOG_DIR;
|
import static org.apache.kafka.server.log.remote.metadata.storage.TopicBasedRemoteLogMetadataManagerConfig.LOG_DIR;
|
||||||
|
|
||||||
@SuppressWarnings("deprecation") // Added for Scala 2.12 compatibility for usages of JavaConverters
|
@ExtendWith(value = ClusterTestExtensions.class)
|
||||||
|
@Tag("integration")
|
||||||
public class TopicBasedRemoteLogMetadataManagerRestartTest {
|
public class TopicBasedRemoteLogMetadataManagerRestartTest {
|
||||||
|
|
||||||
private static final int SEG_SIZE = 1024 * 1024;
|
private static final int SEG_SIZE = 1024 * 1024;
|
||||||
|
|
||||||
private final Time time = new MockTime(1);
|
private final Time time = new MockTime(1);
|
||||||
private final String logDir = TestUtils.tempDirectory("_rlmm_segs_").getAbsolutePath();
|
private final String logDir = TestUtils.tempDirectory("_rlmm_segs_").getAbsolutePath();
|
||||||
|
private final ClusterInstance clusterInstance;
|
||||||
|
|
||||||
private TopicBasedRemoteLogMetadataManagerHarness remoteLogMetadataManagerHarness;
|
TopicBasedRemoteLogMetadataManagerRestartTest(ClusterInstance clusterInstance) { // Constructor injections
|
||||||
|
this.clusterInstance = clusterInstance;
|
||||||
@BeforeEach
|
|
||||||
public void setup() {
|
|
||||||
// Start the cluster and initialize TopicBasedRemoteLogMetadataManager.
|
|
||||||
remoteLogMetadataManagerHarness = new TopicBasedRemoteLogMetadataManagerHarness() {
|
|
||||||
protected Map<String, Object> overrideRemoteLogMetadataManagerProps() {
|
|
||||||
Map<String, Object> props = new HashMap<>();
|
|
||||||
props.put(LOG_DIR, logDir);
|
|
||||||
return props;
|
|
||||||
}
|
|
||||||
};
|
|
||||||
remoteLogMetadataManagerHarness.initialize(Collections.emptySet(), true);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
private void startTopicBasedRemoteLogMetadataManagerHarness(boolean startConsumerThread) {
|
private TopicBasedRemoteLogMetadataManager createTopicBasedRemoteLogMetadataManager() {
|
||||||
remoteLogMetadataManagerHarness.initializeRemoteLogMetadataManager(Collections.emptySet(), startConsumerThread, RemoteLogMetadataTopicPartitioner::new);
|
return RemoteLogMetadataManagerTestUtils.builder()
|
||||||
|
.topicIdPartitions(Collections.emptySet())
|
||||||
|
.bootstrapServers(clusterInstance.bootstrapServers())
|
||||||
|
.startConsumerThread(true)
|
||||||
|
.remoteLogMetadataTopicPartitioner(RemoteLogMetadataTopicPartitioner::new)
|
||||||
|
.overrideRemoteLogMetadataManagerProps(Collections.singletonMap(LOG_DIR, logDir))
|
||||||
|
.build();
|
||||||
}
|
}
|
||||||
|
|
||||||
@AfterEach
|
@ClusterTest(brokers = 3)
|
||||||
public void teardown() throws IOException {
|
|
||||||
if (remoteLogMetadataManagerHarness != null) {
|
|
||||||
remoteLogMetadataManagerHarness.close();
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private void stopTopicBasedRemoteLogMetadataManagerHarness() {
|
|
||||||
remoteLogMetadataManagerHarness.closeRemoteLogMetadataManager();
|
|
||||||
}
|
|
||||||
|
|
||||||
private TopicBasedRemoteLogMetadataManager topicBasedRlmm() {
|
|
||||||
return remoteLogMetadataManagerHarness.remoteLogMetadataManager();
|
|
||||||
}
|
|
||||||
|
|
||||||
@Test
|
|
||||||
public void testRLMMAPIsAfterRestart() throws Exception {
|
public void testRLMMAPIsAfterRestart() throws Exception {
|
||||||
// Create topics.
|
// Create topics.
|
||||||
String leaderTopic = "new-leader";
|
String leaderTopic = "new-leader";
|
||||||
HashMap<Object, Seq<Object>> assignedLeaderTopicReplicas = new HashMap<>();
|
|
||||||
List<Object> leaderTopicReplicas = new ArrayList<>();
|
|
||||||
// Set broker id 0 as the first entry which is taken as the leader.
|
|
||||||
leaderTopicReplicas.add(0);
|
|
||||||
leaderTopicReplicas.add(1);
|
|
||||||
leaderTopicReplicas.add(2);
|
|
||||||
assignedLeaderTopicReplicas.put(0, JavaConverters.asScalaBuffer(leaderTopicReplicas));
|
|
||||||
remoteLogMetadataManagerHarness.createTopicWithAssignment(
|
|
||||||
leaderTopic, JavaConverters.mapAsScalaMap(assignedLeaderTopicReplicas),
|
|
||||||
remoteLogMetadataManagerHarness.listenerName());
|
|
||||||
|
|
||||||
String followerTopic = "new-follower";
|
String followerTopic = "new-follower";
|
||||||
HashMap<Object, Seq<Object>> assignedFollowerTopicReplicas = new HashMap<>();
|
try (Admin admin = clusterInstance.createAdminClient()) {
|
||||||
List<Object> followerTopicReplicas = new ArrayList<>();
|
// Set broker id 0 as the first entry which is taken as the leader.
|
||||||
// Set broker id 1 as the first entry which is taken as the leader.
|
NewTopic newLeaderTopic = new NewTopic(leaderTopic, Collections.singletonMap(0, Arrays.asList(0, 1, 2)));
|
||||||
followerTopicReplicas.add(1);
|
// Set broker id 1 as the first entry which is taken as the leader.
|
||||||
followerTopicReplicas.add(2);
|
NewTopic newFollowerTopic = new NewTopic(followerTopic, Collections.singletonMap(0, Arrays.asList(1, 2, 0)));
|
||||||
followerTopicReplicas.add(0);
|
admin.createTopics(Arrays.asList(newLeaderTopic, newFollowerTopic));
|
||||||
assignedFollowerTopicReplicas.put(0, JavaConverters.asScalaBuffer(followerTopicReplicas));
|
}
|
||||||
remoteLogMetadataManagerHarness.createTopicWithAssignment(followerTopic,
|
|
||||||
JavaConverters.mapAsScalaMap(assignedFollowerTopicReplicas),
|
|
||||||
remoteLogMetadataManagerHarness.listenerName());
|
|
||||||
|
|
||||||
final TopicIdPartition leaderTopicIdPartition = new TopicIdPartition(Uuid.randomUuid(), new TopicPartition(leaderTopic, 0));
|
final TopicIdPartition leaderTopicIdPartition = new TopicIdPartition(Uuid.randomUuid(), new TopicPartition(leaderTopic, 0));
|
||||||
final TopicIdPartition followerTopicIdPartition = new TopicIdPartition(Uuid.randomUuid(), new TopicPartition(followerTopic, 0));
|
final TopicIdPartition followerTopicIdPartition = new TopicIdPartition(Uuid.randomUuid(), new TopicPartition(followerTopic, 0));
|
||||||
|
|
||||||
// Register these partitions to RLMM.
|
|
||||||
topicBasedRlmm().onPartitionLeadershipChanges(Collections.singleton(leaderTopicIdPartition), Collections.singleton(followerTopicIdPartition));
|
|
||||||
|
|
||||||
// Add segments for these partitions, but they are not available as they have not yet been subscribed.
|
|
||||||
RemoteLogSegmentMetadata leaderSegmentMetadata = new RemoteLogSegmentMetadata(
|
RemoteLogSegmentMetadata leaderSegmentMetadata = new RemoteLogSegmentMetadata(
|
||||||
new RemoteLogSegmentId(leaderTopicIdPartition, Uuid.randomUuid()),
|
new RemoteLogSegmentId(leaderTopicIdPartition, Uuid.randomUuid()),
|
||||||
0, 100, -1L, 0,
|
0, 100, -1L, 0,
|
||||||
time.milliseconds(), SEG_SIZE, Collections.singletonMap(0, 0L));
|
time.milliseconds(), SEG_SIZE, Collections.singletonMap(0, 0L));
|
||||||
topicBasedRlmm().addRemoteLogSegmentMetadata(leaderSegmentMetadata).get();
|
|
||||||
|
|
||||||
RemoteLogSegmentMetadata followerSegmentMetadata = new RemoteLogSegmentMetadata(
|
RemoteLogSegmentMetadata followerSegmentMetadata = new RemoteLogSegmentMetadata(
|
||||||
new RemoteLogSegmentId(followerTopicIdPartition, Uuid.randomUuid()),
|
new RemoteLogSegmentId(followerTopicIdPartition, Uuid.randomUuid()),
|
||||||
0, 100, -1L, 0,
|
0, 100, -1L, 0,
|
||||||
time.milliseconds(), SEG_SIZE, Collections.singletonMap(0, 0L));
|
time.milliseconds(), SEG_SIZE, Collections.singletonMap(0, 0L));
|
||||||
topicBasedRlmm().addRemoteLogSegmentMetadata(followerSegmentMetadata).get();
|
|
||||||
|
|
||||||
// Stop TopicBasedRemoteLogMetadataManager only.
|
try (TopicBasedRemoteLogMetadataManager topicBasedRemoteLogMetadataManager = createTopicBasedRemoteLogMetadataManager()) {
|
||||||
stopTopicBasedRemoteLogMetadataManagerHarness();
|
// Register these partitions to RemoteLogMetadataManager.
|
||||||
|
topicBasedRemoteLogMetadataManager.onPartitionLeadershipChanges(
|
||||||
|
Collections.singleton(leaderTopicIdPartition), Collections.singleton(followerTopicIdPartition));
|
||||||
|
|
||||||
// Start TopicBasedRemoteLogMetadataManager
|
// Add segments for these partitions, but they are not available as they have not yet been subscribed.
|
||||||
startTopicBasedRemoteLogMetadataManagerHarness(true);
|
topicBasedRemoteLogMetadataManager.addRemoteLogSegmentMetadata(leaderSegmentMetadata).get();
|
||||||
|
topicBasedRemoteLogMetadataManager.addRemoteLogSegmentMetadata(followerSegmentMetadata).get();
|
||||||
|
}
|
||||||
|
|
||||||
// Register these partitions to RLMM, which loads the respective metadata snapshots.
|
try (TopicBasedRemoteLogMetadataManager topicBasedRemoteLogMetadataManager = createTopicBasedRemoteLogMetadataManager()) {
|
||||||
topicBasedRlmm().onPartitionLeadershipChanges(
|
// Register these partitions to RemoteLogMetadataManager, which loads the respective metadata snapshots.
|
||||||
Collections.singleton(leaderTopicIdPartition), Collections.singleton(followerTopicIdPartition));
|
topicBasedRemoteLogMetadataManager.onPartitionLeadershipChanges(
|
||||||
|
Collections.singleton(leaderTopicIdPartition), Collections.singleton(followerTopicIdPartition));
|
||||||
|
|
||||||
// Check for the stored entries from the earlier run.
|
// Check for the stored entries from the earlier run.
|
||||||
TestUtils.waitForCondition(() ->
|
TestUtils.waitForCondition(() ->
|
||||||
TestUtils.sameElementsWithoutOrder(Collections.singleton(leaderSegmentMetadata).iterator(),
|
TestUtils.sameElementsWithoutOrder(Collections.singleton(leaderSegmentMetadata).iterator(),
|
||||||
topicBasedRlmm().listRemoteLogSegments(leaderTopicIdPartition)),
|
topicBasedRemoteLogMetadataManager.listRemoteLogSegments(leaderTopicIdPartition)),
|
||||||
"Remote log segment metadata not available");
|
"Remote log segment metadata not available");
|
||||||
TestUtils.waitForCondition(() ->
|
TestUtils.waitForCondition(() ->
|
||||||
TestUtils.sameElementsWithoutOrder(Collections.singleton(followerSegmentMetadata).iterator(),
|
TestUtils.sameElementsWithoutOrder(Collections.singleton(followerSegmentMetadata).iterator(),
|
||||||
topicBasedRlmm().listRemoteLogSegments(followerTopicIdPartition)),
|
topicBasedRemoteLogMetadataManager.listRemoteLogSegments(followerTopicIdPartition)),
|
||||||
"Remote log segment metadata not available");
|
"Remote log segment metadata not available");
|
||||||
// Add one more segment
|
// Add one more segment
|
||||||
RemoteLogSegmentMetadata leaderSegmentMetadata2 = new RemoteLogSegmentMetadata(
|
RemoteLogSegmentMetadata leaderSegmentMetadata2 = new RemoteLogSegmentMetadata(
|
||||||
new RemoteLogSegmentId(leaderTopicIdPartition, Uuid.randomUuid()),
|
new RemoteLogSegmentId(leaderTopicIdPartition, Uuid.randomUuid()),
|
||||||
101, 200, -1L, 0,
|
101, 200, -1L, 0,
|
||||||
time.milliseconds(), SEG_SIZE, Collections.singletonMap(0, 101L));
|
time.milliseconds(), SEG_SIZE, Collections.singletonMap(0, 101L));
|
||||||
topicBasedRlmm().addRemoteLogSegmentMetadata(leaderSegmentMetadata2).get();
|
topicBasedRemoteLogMetadataManager.addRemoteLogSegmentMetadata(leaderSegmentMetadata2).get();
|
||||||
|
|
||||||
// Check that both the stored segment and recently added segment are available.
|
// Check that both the stored segment and recently added segment are available.
|
||||||
Assertions.assertTrue(TestUtils.sameElementsWithoutOrder(Arrays.asList(leaderSegmentMetadata, leaderSegmentMetadata2).iterator(),
|
Assertions.assertTrue(TestUtils.sameElementsWithoutOrder(Arrays.asList(leaderSegmentMetadata, leaderSegmentMetadata2).iterator(),
|
||||||
topicBasedRlmm().listRemoteLogSegments(leaderTopicIdPartition)));
|
topicBasedRemoteLogMetadataManager.listRemoteLogSegments(leaderTopicIdPartition)));
|
||||||
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
Loading…
Reference in New Issue