Interface RemoteLogMetadataManager
- All Superinterfaces:
AutoCloseable
,Closeable
,Configurable
This class can be plugged in to Kafka cluster by adding the implementation class as
remote.log.metadata.manager.class.name
property value. There is an inbuilt implementation backed by
topic storage in the local cluster. This is used as the default implementation if
remote.log.metadata.manager.class.name is not configured.
remote.log.metadata.manager.class.path
property is about the class path of the RemoteLogStorageManager
implementation. If specified, the RemoteLogStorageManager implementation and its dependent libraries will be loaded
by a dedicated classloader which searches this class path before the Kafka broker class path. The syntax of this
parameter is same with the standard Java class path string.
remote.log.metadata.manager.listener.name
property is about listener name of the local broker to which
it should get connected if needed by RemoteLogMetadataManager implementation. When this is configured all other
required properties can be passed as properties with prefix of 'remote.log.metadata.manager.listener.
Configurable.configure(Map)
is invoked on this instance.
-
Method Summary
Modifier and TypeMethodDescriptionaddRemoteLogSegmentMetadata
(RemoteLogSegmentMetadata remoteLogSegmentMetadata) This method is used to addRemoteLogSegmentMetadata
asynchronously with the containingRemoteLogSegmentId
intoRemoteLogMetadataManager
.highestOffsetForEpoch
(TopicIdPartition topicIdPartition, int leaderEpoch) Returns the highest log offset of topic partition for the given leader epoch in remote storage.listRemoteLogSegments
(TopicIdPartition topicIdPartition) Returns all the remote log segment metadata of the given topicIdPartition.listRemoteLogSegments
(TopicIdPartition topicIdPartition, int leaderEpoch) Returns iterator of remote log segment metadata, sorted byRemoteLogSegmentMetadata.startOffset()
in ascending order which contains the given leader epoch.void
onPartitionLeadershipChanges
(Set<TopicIdPartition> leaderPartitions, Set<TopicIdPartition> followerPartitions) This method is invoked only when there are changes in leadership of the topic partitions that this broker is responsible for.void
onStopPartitions
(Set<TopicIdPartition> partitions) This method is invoked only when the topic partitions are stopped on this broker.putRemotePartitionDeleteMetadata
(RemotePartitionDeleteMetadata remotePartitionDeleteMetadata) This method is used to update the metadata about remote partition delete event asynchronously.remoteLogSegmentMetadata
(TopicIdPartition topicIdPartition, int epochForOffset, long offset) ReturnsRemoteLogSegmentMetadata
if it exists for the given topic partition containing the offset with the given leader-epoch for the offset, else returnsOptional.empty()
.updateRemoteLogSegmentMetadata
(RemoteLogSegmentMetadataUpdate remoteLogSegmentMetadataUpdate) This method is used to update theRemoteLogSegmentMetadata
asynchronously.Methods inherited from interface org.apache.kafka.common.Configurable
configure
-
Method Details
-
addRemoteLogSegmentMetadata
CompletableFuture<Void> addRemoteLogSegmentMetadata(RemoteLogSegmentMetadata remoteLogSegmentMetadata) throws RemoteStorageException This method is used to addRemoteLogSegmentMetadata
asynchronously with the containingRemoteLogSegmentId
intoRemoteLogMetadataManager
.RemoteLogSegmentMetadata is identified by RemoteLogSegmentId and it should have the initial state which is
RemoteLogSegmentState.COPY_SEGMENT_STARTED
.updateRemoteLogSegmentMetadata(RemoteLogSegmentMetadataUpdate)
should be used to update an existing RemoteLogSegmentMetadata.- Parameters:
remoteLogSegmentMetadata
- metadata about the remote log segment.- Returns:
- a CompletableFuture which will complete once this operation is finished.
- Throws:
RemoteStorageException
- if there are any storage related errors occurred.IllegalArgumentException
- if the given metadata instance does not have the state asRemoteLogSegmentState.COPY_SEGMENT_STARTED
-
updateRemoteLogSegmentMetadata
CompletableFuture<Void> updateRemoteLogSegmentMetadata(RemoteLogSegmentMetadataUpdate remoteLogSegmentMetadataUpdate) throws RemoteStorageException This method is used to update theRemoteLogSegmentMetadata
asynchronously. Currently, it allows to update with the new state based on the life cycle of the segment. It can go through the below state transitions.+---------------------+ +----------------------+ |COPY_SEGMENT_STARTED |----------->|COPY_SEGMENT_FINISHED | +-------------------+-+ +--+-------------------+ | | | | v v +--+-----------------+-+ |DELETE_SEGMENT_STARTED| +-----------+----------+ | | v +-----------+-----------+ |DELETE_SEGMENT_FINISHED| +-----------------------+
RemoteLogSegmentState.COPY_SEGMENT_STARTED
- This state indicates that the segment copying to remote storage is started but not yet finished.RemoteLogSegmentState.COPY_SEGMENT_FINISHED
- This state indicates that the segment copying to remote storage is finished.
The leader broker copies the log segments to the remote storage and puts the remote log segment metadata with the state as “COPY_SEGMENT_STARTED” and updates the state as “COPY_SEGMENT_FINISHED” once the copy is successful.RemoteLogSegmentState.DELETE_SEGMENT_STARTED
- This state indicates that the segment deletion is started but not yet finished.RemoteLogSegmentState.DELETE_SEGMENT_FINISHED
- This state indicates that the segment is deleted successfully.
Leader partitions publish both the above delete segment events when remote log retention is reached for the respective segments. Remote Partition Removers also publish these events when a segment is deleted as part of the remote partition deletion.- Parameters:
remoteLogSegmentMetadataUpdate
- update of the remote log segment metadata.- Returns:
- a CompletableFuture which will complete once this operation is finished.
- Throws:
RemoteStorageException
- if there are any storage related errors occurred.RemoteResourceNotFoundException
- when there are no resources associated with the given remoteLogSegmentMetadataUpdate.IllegalArgumentException
- if the given metadata instance has the state asRemoteLogSegmentState.COPY_SEGMENT_STARTED
-
remoteLogSegmentMetadata
Optional<RemoteLogSegmentMetadata> remoteLogSegmentMetadata(TopicIdPartition topicIdPartition, int epochForOffset, long offset) throws RemoteStorageException ReturnsRemoteLogSegmentMetadata
if it exists for the given topic partition containing the offset with the given leader-epoch for the offset, else returnsOptional.empty()
.- Parameters:
topicIdPartition
- topic partitionepochForOffset
- leader epoch for the given offsetoffset
- offset- Returns:
- the requested remote log segment metadata if it exists.
- Throws:
RemoteStorageException
- if there are any storage related errors occurred.
-
highestOffsetForEpoch
Optional<Long> highestOffsetForEpoch(TopicIdPartition topicIdPartition, int leaderEpoch) throws RemoteStorageException Returns the highest log offset of topic partition for the given leader epoch in remote storage. This is used by remote log management subsystem to know up to which offset the segments have been copied to remote storage for a given leader epoch.- Parameters:
topicIdPartition
- topic partitionleaderEpoch
- leader epoch- Returns:
- the requested highest log offset if exists.
- Throws:
RemoteStorageException
- if there are any storage related errors occurred.
-
putRemotePartitionDeleteMetadata
CompletableFuture<Void> putRemotePartitionDeleteMetadata(RemotePartitionDeleteMetadata remotePartitionDeleteMetadata) throws RemoteStorageException This method is used to update the metadata about remote partition delete event asynchronously. Currently, it allows updating the state (RemotePartitionDeleteState
) of a topic partition in remote metadata storage. Controller invokes this method withRemotePartitionDeleteMetadata
having state asRemotePartitionDeleteState.DELETE_PARTITION_MARKED
. So, remote partition removers can act on this event to clean the respective remote log segments of the partition.
In the case of default RLMM implementation, remote partition remover processesRemotePartitionDeleteState.DELETE_PARTITION_MARKED
- sends an event with state as
RemotePartitionDeleteState.DELETE_PARTITION_STARTED
- gets all the remote log segments and deletes them.
- sends an event with state as
RemotePartitionDeleteState.DELETE_PARTITION_FINISHED
once all the remote log segments are deleted.
- Parameters:
remotePartitionDeleteMetadata
- update on delete state of a partition.- Returns:
- a CompletableFuture which will complete once this operation is finished.
- Throws:
RemoteStorageException
- if there are any storage related errors occurred.RemoteResourceNotFoundException
- when there are no resources associated with the given remotePartitionDeleteMetadata.
- sends an event with state as
-
listRemoteLogSegments
Iterator<RemoteLogSegmentMetadata> listRemoteLogSegments(TopicIdPartition topicIdPartition) throws RemoteStorageException Returns all the remote log segment metadata of the given topicIdPartition.Remote Partition Removers uses this method to fetch all the segments for a given topic partition, so that they can delete them.
- Returns:
- Iterator of all the remote log segment metadata for the given topic partition.
- Throws:
RemoteStorageException
-
listRemoteLogSegments
Iterator<RemoteLogSegmentMetadata> listRemoteLogSegments(TopicIdPartition topicIdPartition, int leaderEpoch) throws RemoteStorageException Returns iterator of remote log segment metadata, sorted byRemoteLogSegmentMetadata.startOffset()
in ascending order which contains the given leader epoch. This is used by remote log retention management subsystem to fetch the segment metadata for a given leader epoch.- Parameters:
topicIdPartition
- topic partitionleaderEpoch
- leader epoch- Returns:
- Iterator of remote segments, sorted by start offset in ascending order.
- Throws:
RemoteStorageException
-
onPartitionLeadershipChanges
void onPartitionLeadershipChanges(Set<TopicIdPartition> leaderPartitions, Set<TopicIdPartition> followerPartitions) This method is invoked only when there are changes in leadership of the topic partitions that this broker is responsible for.- Parameters:
leaderPartitions
- partitions that have become leaders on this broker.followerPartitions
- partitions that have become followers on this broker.
-
onStopPartitions
This method is invoked only when the topic partitions are stopped on this broker. This can happen when a partition is emigrated to other broker or a partition is deleted.- Parameters:
partitions
- topic partitions that have been stopped.
-