@InterfaceStability.Evolving public interface RemoteLogMetadataManager extends Configurable, Closeable
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 RemoteLogMetadataManager
implementation. If specified, the RemoteLogMetadataManager 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.
Configurable.configure(Map)
is invoked on this instance.
Modifier and Type | Method and Description |
---|---|
CompletableFuture<Void> |
addRemoteLogSegmentMetadata(RemoteLogSegmentMetadata remoteLogSegmentMetadata)
This method is used to add
RemoteLogSegmentMetadata asynchronously with the containing RemoteLogSegmentId into RemoteLogMetadataManager . |
Optional<Long> |
highestOffsetForEpoch(TopicIdPartition topicIdPartition,
int leaderEpoch)
Returns the highest log offset of topic partition for the given leader epoch in remote storage.
|
Iterator<RemoteLogSegmentMetadata> |
listRemoteLogSegments(TopicIdPartition topicIdPartition)
Returns all the remote log segment metadata of the given topicIdPartition.
|
Iterator<RemoteLogSegmentMetadata> |
listRemoteLogSegments(TopicIdPartition topicIdPartition,
int leaderEpoch)
Returns iterator of remote log segment metadata, sorted by
RemoteLogSegmentMetadata.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.
|
CompletableFuture<Void> |
putRemotePartitionDeleteMetadata(RemotePartitionDeleteMetadata remotePartitionDeleteMetadata)
This method is used to update the metadata about remote partition delete event asynchronously.
|
Optional<RemoteLogSegmentMetadata> |
remoteLogSegmentMetadata(TopicIdPartition topicIdPartition,
int epochForOffset,
long offset)
Returns
RemoteLogSegmentMetadata if it exists for the given topic partition containing the offset with
the given leader-epoch for the offset, else returns Optional.empty() . |
long |
remoteLogSize(TopicIdPartition topicIdPartition,
int leaderEpoch)
Returns total size of the log for the given leader epoch in remote storage.
|
CompletableFuture<Void> |
updateRemoteLogSegmentMetadata(RemoteLogSegmentMetadataUpdate remoteLogSegmentMetadataUpdate)
This method is used to update the
RemoteLogSegmentMetadata asynchronously. |
configure
CompletableFuture<Void> addRemoteLogSegmentMetadata(RemoteLogSegmentMetadata remoteLogSegmentMetadata) throws RemoteStorageException
RemoteLogSegmentMetadata
asynchronously with the containing RemoteLogSegmentId
into RemoteLogMetadataManager
.
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.
remoteLogSegmentMetadata
- metadata about the remote log segment.RemoteStorageException
- if there are any storage related errors occurred.IllegalArgumentException
- if the given metadata instance does not have the state as RemoteLogSegmentState.COPY_SEGMENT_STARTED
CompletableFuture<Void> updateRemoteLogSegmentMetadata(RemoteLogSegmentMetadataUpdate remoteLogSegmentMetadataUpdate) throws RemoteStorageException
RemoteLogSegmentMetadata
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.
remoteLogSegmentMetadataUpdate
- update of the remote log segment metadata.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 as RemoteLogSegmentState.COPY_SEGMENT_STARTED
Optional<RemoteLogSegmentMetadata> remoteLogSegmentMetadata(TopicIdPartition topicIdPartition, int epochForOffset, long offset) throws RemoteStorageException
RemoteLogSegmentMetadata
if it exists for the given topic partition containing the offset with
the given leader-epoch for the offset, else returns Optional.empty()
.topicIdPartition
- topic partitionepochForOffset
- leader epoch for the given offsetoffset
- offsetRemoteStorageException
- if there are any storage related errors occurred.Optional<Long> highestOffsetForEpoch(TopicIdPartition topicIdPartition, int leaderEpoch) throws RemoteStorageException
topicIdPartition
- topic partitionleaderEpoch
- leader epochRemoteStorageException
- if there are any storage related errors occurred.CompletableFuture<Void> putRemotePartitionDeleteMetadata(RemotePartitionDeleteMetadata remotePartitionDeleteMetadata) throws RemoteStorageException
RemotePartitionDeleteState
) of a topic partition in remote metadata storage. Controller invokes
this method with RemotePartitionDeleteMetadata
having state as RemotePartitionDeleteState.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 processes RemotePartitionDeleteState.DELETE_PARTITION_MARKED
RemotePartitionDeleteState.DELETE_PARTITION_STARTED
RemotePartitionDeleteState.DELETE_PARTITION_FINISHED
once all the remote log segments are
deleted.
remotePartitionDeleteMetadata
- update on delete state of a partition.RemoteStorageException
- if there are any storage related errors occurred.RemoteResourceNotFoundException
- when there are no resources associated with the given remotePartitionDeleteMetadata.Iterator<RemoteLogSegmentMetadata> listRemoteLogSegments(TopicIdPartition topicIdPartition) throws RemoteStorageException
Remote Partition Removers uses this method to fetch all the segments for a given topic partition, so that they can delete them.
RemoteStorageException
Iterator<RemoteLogSegmentMetadata> listRemoteLogSegments(TopicIdPartition topicIdPartition, int leaderEpoch) throws RemoteStorageException
RemoteLogSegmentMetadata.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.topicIdPartition
- topic partitionleaderEpoch
- leader epochRemoteStorageException
void onPartitionLeadershipChanges(Set<TopicIdPartition> leaderPartitions, Set<TopicIdPartition> followerPartitions)
leaderPartitions
- partitions that have become leaders on this broker.followerPartitions
- partitions that have become followers on this broker.void onStopPartitions(Set<TopicIdPartition> partitions)
partitions
- topic partitions that have been stopped.long remoteLogSize(TopicIdPartition topicIdPartition, int leaderEpoch) throws RemoteStorageException
topicIdPartition
- topic partition for which size needs to be calculated.leaderEpoch
- Size will only include segments belonging to this epoch.RemoteStorageException