Package org.apache.kafka.connect.mirror
Class MirrorClient
- java.lang.Object
-
- org.apache.kafka.connect.mirror.MirrorClient
-
- All Implemented Interfaces:
AutoCloseable
public class MirrorClient extends Object implements AutoCloseable
Interprets MM2's internal topics (checkpoints, heartbeats) on a given cluster.Given a top-level "mm2.properties" configuration file, MirrorClients can be constructed for individual clusters as follows:
MirrorMakerConfig mmConfig = new MirrorMakerConfig(props); MirrorClientConfig mmClientConfig = mmConfig.clientConfig("some-cluster"); MirrorClient mmClient = new Mirrorclient(mmClientConfig);
-
-
Constructor Summary
Constructors Constructor Description MirrorClient(Map<String,Object> props)
MirrorClient(MirrorClientConfig config)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description Set<String>
checkpointTopics()
Find all checkpoint topics on this cluster.void
close()
Close internal clients.Set<String>
heartbeatTopics()
Find all heartbeat topics on this cluster.Map<TopicPartition,OffsetAndMetadata>
remoteConsumerOffsets(String consumerGroupId, String remoteClusterAlias, Duration timeout)
Translate a remote consumer group's offsets into corresponding local offsets.Set<String>
remoteTopics()
Find all remote topics on this cluster.Set<String>
remoteTopics(String source)
Find all remote topics that have been replicated directly from the given source cluster.int
replicationHops(String upstreamClusterAlias)
Compute shortest number of hops from an upstream source cluster.ReplicationPolicy
replicationPolicy()
Get the ReplicationPolicy instance used to interpret remote topics.Set<String>
upstreamClusters()
Find upstream clusters, which may be multiple hops away, based on incoming heartbeats.
-
-
-
Constructor Detail
-
MirrorClient
public MirrorClient(MirrorClientConfig config)
-
-
Method Detail
-
close
public void close()
Close internal clients.- Specified by:
close
in interfaceAutoCloseable
-
replicationPolicy
public ReplicationPolicy replicationPolicy()
Get the ReplicationPolicy instance used to interpret remote topics. This instance is constructed based on relevant configuration properties, includingreplication.policy.class
.
-
replicationHops
public int replicationHops(String upstreamClusterAlias) throws InterruptedException
Compute shortest number of hops from an upstream source cluster. For example, given replication flow A->B->C, there are two hops from A to C. Returns -1 if upstream cluster is unreachable.- Throws:
InterruptedException
-
heartbeatTopics
public Set<String> heartbeatTopics() throws InterruptedException
Find all heartbeat topics on this cluster. Heartbeat topics are replicated from other clusters.- Throws:
InterruptedException
-
checkpointTopics
public Set<String> checkpointTopics() throws InterruptedException
Find all checkpoint topics on this cluster.- Throws:
InterruptedException
-
upstreamClusters
public Set<String> upstreamClusters() throws InterruptedException
Find upstream clusters, which may be multiple hops away, based on incoming heartbeats.- Throws:
InterruptedException
-
remoteTopics
public Set<String> remoteTopics() throws InterruptedException
Find all remote topics on this cluster. This does not include internal topics (heartbeats, checkpoints).- Throws:
InterruptedException
-
remoteTopics
public Set<String> remoteTopics(String source) throws InterruptedException
Find all remote topics that have been replicated directly from the given source cluster.- Throws:
InterruptedException
-
remoteConsumerOffsets
public Map<TopicPartition,OffsetAndMetadata> remoteConsumerOffsets(String consumerGroupId, String remoteClusterAlias, Duration timeout)
Translate a remote consumer group's offsets into corresponding local offsets. Topics are automatically renamed according to the ReplicationPolicy.- Parameters:
consumerGroupId
- group ID of remote consumer groupremoteClusterAlias
- alias of remote clustertimeout
- timeout
-
-