Interface Admin

All Superinterfaces:
AutoCloseable
All Known Implementing Classes:
AdminClient, KafkaAdminClient

@Evolving public interface Admin extends AutoCloseable
The administrative client for Kafka, which supports managing and inspecting topics, brokers, configurations and ACLs.

Instances returned from the create methods of this interface are guaranteed to be thread safe. However, the KafkaFutures returned from request methods are executed by a single thread so it is important that any code which executes on that thread when they complete (using KafkaFuture.thenApply(KafkaFuture.Function), for example) doesn't block for too long. If necessary, processing of results should be passed to another thread.

The operations exposed by Admin follow a consistent pattern:

  • Admin instances should be created using create(Properties) or create(Map)
  • Each operation typically has two overloaded methods, one which uses a default set of options and an overloaded method where the last parameter is an explicit options object.
  • The operation method's first parameter is a Collection of items to perform the operation on. Batching multiple requests into a single call is more efficient and should be preferred over multiple calls to the same method.
  • The operation methods execute asynchronously.
  • Each xxx operation method returns an XxxResult class with methods which expose KafkaFuture for accessing the result(s) of the operation.
  • Typically an all() method is provided for getting the overall success/failure of the batch and a values() method provided access to each item in a request batch. Other methods may also be provided.
  • For synchronous behaviour use KafkaFuture.get()

Here is a simple example of using an Admin client instance to create a new topic:

 
 Properties props = new Properties();
 props.put(AdminClientConfig.BOOTSTRAP_SERVERS_CONFIG, "localhost:9092");

 try (Admin admin = Admin.create(props)) {
   String topicName = "my-topic";
   int partitions = 12;
   short replicationFactor = 3;
   // Create a compacted topic
   CreateTopicsResult result = admin.createTopics(Collections.singleton(
     new NewTopic(topicName, partitions, replicationFactor)
       .configs(Collections.singletonMap(TopicConfig.CLEANUP_POLICY_CONFIG, TopicConfig.CLEANUP_POLICY_COMPACT))));

   // Call values() to get the result for a specific topic
   KafkaFuture<Void> future = result.values().get(topicName);

   // Call get() to block until the topic creation is complete or has failed
   // if creation failed the ExecutionException wraps the underlying cause.
   future.get();
 }
 
 

Bootstrap and balancing

The bootstrap.servers config in the Map or Properties passed to create(Properties) is only used for discovering the brokers in the cluster, which the client will then connect to as needed. As such, it is sufficient to include only two or three broker addresses to cope with the possibility of brokers being unavailable.

Different operations necessitate requests being sent to different nodes in the cluster. For example createTopics(Collection) communicates with the controller, but describeTopics(Collection) can talk to any broker. When the recipient does not matter the instance will try to use the broker with the fewest outstanding requests.

The client will transparently retry certain errors which are usually transient. For example if the request for createTopics() get sent to a node which was not the controller the metadata would be refreshed and the request re-sent to the controller.

Broker Compatibility

The minimum broker version required is 0.10.0.0. Methods with stricter requirements will specify the minimum broker version required.

This client was introduced in 0.11.0.0 and the API is still evolving. We will try to evolve the API in a compatible manner, but we reserve the right to make breaking changes in minor releases, if necessary. We will update the InterfaceStability annotation and this notice once the API is considered stable.

  • Method Details

    • create

      static Admin create(Properties props)
      Create a new Admin with the given configuration.
      Parameters:
      props - The configuration.
      Returns:
      The new KafkaAdminClient.
    • create

      static Admin create(Map<String,Object> conf)
      Create a new Admin with the given configuration.
      Parameters:
      conf - The configuration.
      Returns:
      The new KafkaAdminClient.
    • close

      default void close()
      Close the Admin and release all associated resources.

      See close(Duration)

      Specified by:
      close in interface AutoCloseable
    • close

      void close(Duration timeout)
      Close the Admin client and release all associated resources.

      The close operation has a grace period during which current operations will be allowed to complete, specified by the given duration. New operations will not be accepted during the grace period. Once the grace period is over, all operations that have not yet been completed will be aborted with a TimeoutException.

      Parameters:
      timeout - The time to use for the wait time.
    • createTopics

      default CreateTopicsResult createTopics(Collection<NewTopic> newTopics)
      Create a batch of new topics with the default options.

      This is a convenience method for createTopics(Collection, CreateTopicsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.10.1.0 or higher.

      Parameters:
      newTopics - The new topics to create.
      Returns:
      The CreateTopicsResult.
    • createTopics

      CreateTopicsResult createTopics(Collection<NewTopic> newTopics, CreateTopicsOptions options)
      Create a batch of new topics.

      This operation is not transactional so it may succeed for some topics while fail for others.

      It may take several seconds after CreateTopicsResult returns success for all the brokers to become aware that the topics have been created. During this time, listTopics() and describeTopics(Collection) may not return information about the new topics.

      This operation is supported by brokers with version 0.10.1.0 or higher. The validateOnly option is supported from version 0.10.2.0.

      Parameters:
      newTopics - The new topics to create.
      options - The options to use when creating the new topics.
      Returns:
      The CreateTopicsResult.
    • deleteTopics

      default DeleteTopicsResult deleteTopics(Collection<String> topics)
      This is a convenience method for deleteTopics(TopicCollection, DeleteTopicsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.10.1.0 or higher.

      Parameters:
      topics - The topic names to delete.
      Returns:
      The DeleteTopicsResult.
    • deleteTopics

      default DeleteTopicsResult deleteTopics(Collection<String> topics, DeleteTopicsOptions options)
      This is a convenience method for deleteTopics(TopicCollection, DeleteTopicsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.10.1.0 or higher.

      Parameters:
      topics - The topic names to delete.
      options - The options to use when deleting the topics.
      Returns:
      The DeleteTopicsResult.
    • deleteTopics

      default DeleteTopicsResult deleteTopics(TopicCollection topics)
      This is a convenience method for deleteTopics(TopicCollection, DeleteTopicsOptions) with default options. See the overload for more details.

      When using topic IDs, this operation is supported by brokers with inter-broker protocol 2.8 or higher. When using topic names, this operation is supported by brokers with version 0.10.1.0 or higher.

      Parameters:
      topics - The topics to delete.
      Returns:
      The DeleteTopicsResult.
    • deleteTopics

      DeleteTopicsResult deleteTopics(TopicCollection topics, DeleteTopicsOptions options)
      Delete a batch of topics.

      This operation is not transactional so it may succeed for some topics while fail for others.

      It may take several seconds after the DeleteTopicsResult returns success for all the brokers to become aware that the topics are gone. During this time, listTopics() and describeTopics(Collection) may continue to return information about the deleted topics.

      If delete.topic.enable is false on the brokers, deleteTopics will mark the topics for deletion, but not actually delete them. The futures will return successfully in this case.

      When using topic IDs, this operation is supported by brokers with inter-broker protocol 2.8 or higher. When using topic names, this operation is supported by brokers with version 0.10.1.0 or higher.

      Parameters:
      topics - The topics to delete.
      options - The options to use when deleting the topics.
      Returns:
      The DeleteTopicsResult.
    • listTopics

      default ListTopicsResult listTopics()
      List the topics available in the cluster with the default options.

      This is a convenience method for listTopics(ListTopicsOptions) with default options. See the overload for more details.

      Returns:
      The ListTopicsResult.
    • listTopics

      ListTopicsResult listTopics(ListTopicsOptions options)
      List the topics available in the cluster.
      Parameters:
      options - The options to use when listing the topics.
      Returns:
      The ListTopicsResult.
    • describeTopics

      default DescribeTopicsResult describeTopics(Collection<String> topicNames)
      Describe some topics in the cluster, with the default options.

      This is a convenience method for describeTopics(Collection, DescribeTopicsOptions) with default options. See the overload for more details.

      Parameters:
      topicNames - The names of the topics to describe.
      Returns:
      The DescribeTopicsResult.
    • describeTopics

      default DescribeTopicsResult describeTopics(Collection<String> topicNames, DescribeTopicsOptions options)
      Describe some topics in the cluster.
      Parameters:
      topicNames - The names of the topics to describe.
      options - The options to use when describing the topic.
      Returns:
      The DescribeTopicsResult.
    • describeTopics

      default DescribeTopicsResult describeTopics(TopicCollection topics)
      This is a convenience method for describeTopics(TopicCollection, DescribeTopicsOptions) with default options. See the overload for more details.

      When using topic IDs, this operation is supported by brokers with version 3.1.0 or higher.

      Parameters:
      topics - The topics to describe.
      Returns:
      The DescribeTopicsResult.
    • describeTopics

      Describe some topics in the cluster. When using topic IDs, this operation is supported by brokers with version 3.1.0 or higher.
      Parameters:
      topics - The topics to describe.
      options - The options to use when describing the topics.
      Returns:
      The DescribeTopicsResult.
    • describeCluster

      default DescribeClusterResult describeCluster()
      Get information about the nodes in the cluster, using the default options.

      This is a convenience method for describeCluster(DescribeClusterOptions) with default options. See the overload for more details.

      Returns:
      The DescribeClusterResult.
    • describeCluster

      Get information about the nodes in the cluster.
      Parameters:
      options - The options to use when getting information about the cluster.
      Returns:
      The DescribeClusterResult.
    • describeAcls

      default DescribeAclsResult describeAcls(AclBindingFilter filter)
      This is a convenience method for describeAcls(AclBindingFilter, DescribeAclsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      filter - The filter to use.
      Returns:
      The DescribeAclsResult.
    • describeAcls

      Lists access control lists (ACLs) according to the supplied filter.

      Note: it may take some time for changes made by createAcls or deleteAcls to be reflected in the output of describeAcls.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      filter - The filter to use.
      options - The options to use when listing the ACLs.
      Returns:
      The DescribeAclsResult.
    • createAcls

      default CreateAclsResult createAcls(Collection<AclBinding> acls)
      This is a convenience method for createAcls(Collection, CreateAclsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      acls - The ACLs to create
      Returns:
      The CreateAclsResult.
    • createAcls

      Creates access control lists (ACLs) which are bound to specific resources.

      This operation is not transactional so it may succeed for some ACLs while fail for others.

      If you attempt to add an ACL that duplicates an existing ACL, no error will be raised, but no changes will be made.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      acls - The ACLs to create
      options - The options to use when creating the ACLs.
      Returns:
      The CreateAclsResult.
    • deleteAcls

      default DeleteAclsResult deleteAcls(Collection<AclBindingFilter> filters)
      This is a convenience method for deleteAcls(Collection, DeleteAclsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      filters - The filters to use.
      Returns:
      The DeleteAclsResult.
    • deleteAcls

      Deletes access control lists (ACLs) according to the supplied filters.

      This operation is not transactional so it may succeed for some ACLs while fail for others.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      filters - The filters to use.
      options - The options to use when deleting the ACLs.
      Returns:
      The DeleteAclsResult.
    • describeConfigs

      default DescribeConfigsResult describeConfigs(Collection<ConfigResource> resources)
      Get the configuration for the specified resources with the default options.

      This is a convenience method for describeConfigs(Collection, DescribeConfigsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      resources - The resources (topic and broker resource types are currently supported)
      Returns:
      The DescribeConfigsResult
    • describeConfigs

      Get the configuration for the specified resources.

      The returned configuration includes default values and the isDefault() method can be used to distinguish them from user supplied values.

      The value of config entries where isSensitive() is true is always null so that sensitive information is not disclosed.

      Config entries where isReadOnly() is true cannot be updated.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      resources - The resources (topic and broker resource types are currently supported)
      options - The options to use when describing configs
      Returns:
      The DescribeConfigsResult
    • alterConfigs

      @Deprecated default AlterConfigsResult alterConfigs(Map<ConfigResource,Config> configs)
      Deprecated.
      Update the configuration for the specified resources with the default options.

      This is a convenience method for alterConfigs(Map, AlterConfigsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      configs - The resources with their configs (topic is the only resource type with configs that can be updated currently)
      Returns:
      The AlterConfigsResult
    • alterConfigs

      Update the configuration for the specified resources with the default options.

      Updates are not transactional so they may succeed for some resources while fail for others. The configs for a particular resource are updated atomically.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      configs - The resources with their configs (topic is the only resource type with configs that can be updated currently)
      options - The options to use when describing configs
      Returns:
      The AlterConfigsResult
    • incrementalAlterConfigs

      default AlterConfigsResult incrementalAlterConfigs(Map<ConfigResource,Collection<AlterConfigOp>> configs)
      Incrementally updates the configuration for the specified resources with default options.

      This is a convenience method for incrementalAlterConfigs(Map, AlterConfigsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 2.3.0 or higher.

      Parameters:
      configs - The resources with their configs
      Returns:
      The AlterConfigsResult
    • incrementalAlterConfigs

      AlterConfigsResult incrementalAlterConfigs(Map<ConfigResource,Collection<AlterConfigOp>> configs, AlterConfigsOptions options)
      Incrementally update the configuration for the specified resources.

      Updates are not transactional so they may succeed for some resources while fail for others. The configs for a particular resource are updated atomically.

      The following exceptions can be anticipated when calling get() on the futures obtained from the returned AlterConfigsResult:

      This operation is supported by brokers with version 2.3.0 or higher.

      Parameters:
      configs - The resources with their configs
      options - The options to use when altering configs
      Returns:
      The AlterConfigsResult
    • alterReplicaLogDirs

      default AlterReplicaLogDirsResult alterReplicaLogDirs(Map<TopicPartitionReplica,String> replicaAssignment)
      Change the log directory for the specified replicas. If the replica does not exist on the broker, the result shows REPLICA_NOT_AVAILABLE for the given replica and the replica will be created in the given log directory on the broker when it is created later. If the replica already exists on the broker, the replica will be moved to the given log directory if it is not already there. For detailed result, inspect the returned AlterReplicaLogDirsResult instance.

      This operation is not transactional so it may succeed for some replicas while fail for others.

      This is a convenience method for alterReplicaLogDirs(Map, AlterReplicaLogDirsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 1.1.0 or higher.

      Parameters:
      replicaAssignment - The replicas with their log directory absolute path
      Returns:
      The AlterReplicaLogDirsResult
    • alterReplicaLogDirs

      AlterReplicaLogDirsResult alterReplicaLogDirs(Map<TopicPartitionReplica,String> replicaAssignment, AlterReplicaLogDirsOptions options)
      Change the log directory for the specified replicas. If the replica does not exist on the broker, the result shows REPLICA_NOT_AVAILABLE for the given replica and the replica will be created in the given log directory on the broker when it is created later. If the replica already exists on the broker, the replica will be moved to the given log directory if it is not already there. For detailed result, inspect the returned AlterReplicaLogDirsResult instance.

      This operation is not transactional so it may succeed for some replicas while fail for others.

      This operation is supported by brokers with version 1.1.0 or higher.

      Parameters:
      replicaAssignment - The replicas with their log directory absolute path
      options - The options to use when changing replica dir
      Returns:
      The AlterReplicaLogDirsResult
    • describeLogDirs

      default DescribeLogDirsResult describeLogDirs(Collection<Integer> brokers)
      Query the information of all log directories on the given set of brokers

      This is a convenience method for describeLogDirs(Collection, DescribeLogDirsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 1.0.0 or higher.

      Parameters:
      brokers - A list of brokers
      Returns:
      The DescribeLogDirsResult
    • describeLogDirs

      DescribeLogDirsResult describeLogDirs(Collection<Integer> brokers, DescribeLogDirsOptions options)
      Query the information of all log directories on the given set of brokers

      This operation is supported by brokers with version 1.0.0 or higher.

      Parameters:
      brokers - A list of brokers
      options - The options to use when querying log dir info
      Returns:
      The DescribeLogDirsResult
    • describeReplicaLogDirs

      default DescribeReplicaLogDirsResult describeReplicaLogDirs(Collection<TopicPartitionReplica> replicas)
      Query the replica log directory information for the specified replicas.

      This is a convenience method for describeReplicaLogDirs(Collection, DescribeReplicaLogDirsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 1.0.0 or higher.

      Parameters:
      replicas - The replicas to query
      Returns:
      The DescribeReplicaLogDirsResult
    • describeReplicaLogDirs

      Query the replica log directory information for the specified replicas.

      This operation is supported by brokers with version 1.0.0 or higher.

      Parameters:
      replicas - The replicas to query
      options - The options to use when querying replica log dir info
      Returns:
      The DescribeReplicaLogDirsResult
    • createPartitions

      default CreatePartitionsResult createPartitions(Map<String,NewPartitions> newPartitions)
      Increase the number of partitions of the topics given as the keys of newPartitions according to the corresponding values. If partitions are increased for a topic that has a key, the partition logic or ordering of the messages will be affected.

      This is a convenience method for createPartitions(Map, CreatePartitionsOptions) with default options. See the overload for more details.

      Parameters:
      newPartitions - The topics which should have new partitions created, and corresponding parameters for the created partitions.
      Returns:
      The CreatePartitionsResult.
    • createPartitions

      CreatePartitionsResult createPartitions(Map<String,NewPartitions> newPartitions, CreatePartitionsOptions options)
      Increase the number of partitions of the topics given as the keys of newPartitions according to the corresponding values. If partitions are increased for a topic that has a key, the partition logic or ordering of the messages will be affected.

      This operation is not transactional so it may succeed for some topics while fail for others.

      It may take several seconds after this method returns success for all the brokers to become aware that the partitions have been created. During this time, describeTopics(Collection) may not return information about the new partitions.

      This operation is supported by brokers with version 1.0.0 or higher.

      The following exceptions can be anticipated when calling get() on the futures obtained from the values() method of the returned CreatePartitionsResult

      Parameters:
      newPartitions - The topics which should have new partitions created, and corresponding parameters for the created partitions.
      options - The options to use when creating the new partitions.
      Returns:
      The CreatePartitionsResult.
    • deleteRecords

      default DeleteRecordsResult deleteRecords(Map<TopicPartition,RecordsToDelete> recordsToDelete)
      Delete records whose offset is smaller than the given offset of the corresponding partition.

      This is a convenience method for deleteRecords(Map, DeleteRecordsOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      recordsToDelete - The topic partitions and related offsets from which records deletion starts.
      Returns:
      The DeleteRecordsResult.
    • deleteRecords

      DeleteRecordsResult deleteRecords(Map<TopicPartition,RecordsToDelete> recordsToDelete, DeleteRecordsOptions options)
      Delete records whose offset is smaller than the given offset of the corresponding partition.

      This operation is supported by brokers with version 0.11.0.0 or higher.

      Parameters:
      recordsToDelete - The topic partitions and related offsets from which records deletion starts.
      options - The options to use when deleting records.
      Returns:
      The DeleteRecordsResult.
    • createDelegationToken

      default CreateDelegationTokenResult createDelegationToken()
      Create a Delegation Token.

      This is a convenience method for createDelegationToken(CreateDelegationTokenOptions) with default options. See the overload for more details.

      Returns:
      The CreateDelegationTokenResult.
    • createDelegationToken

      Create a Delegation Token.

      This operation is supported by brokers with version 1.1.0 or higher.

      The following exceptions can be anticipated when calling get() on the futures obtained from the delegationToken() method of the returned CreateDelegationTokenResult

      Parameters:
      options - The options to use when creating delegation token.
      Returns:
      The DeleteRecordsResult.
    • renewDelegationToken

      default RenewDelegationTokenResult renewDelegationToken(byte[] hmac)
      Renew a Delegation Token.

      This is a convenience method for renewDelegationToken(byte[], RenewDelegationTokenOptions) with default options. See the overload for more details.

      Parameters:
      hmac - HMAC of the Delegation token
      Returns:
      The RenewDelegationTokenResult.
    • renewDelegationToken

      RenewDelegationTokenResult renewDelegationToken(byte[] hmac, RenewDelegationTokenOptions options)
      Renew a Delegation Token.

      This operation is supported by brokers with version 1.1.0 or higher.

      The following exceptions can be anticipated when calling get() on the futures obtained from the expiryTimestamp() method of the returned RenewDelegationTokenResult

      Parameters:
      hmac - HMAC of the Delegation token
      options - The options to use when renewing delegation token.
      Returns:
      The RenewDelegationTokenResult.
    • expireDelegationToken

      default ExpireDelegationTokenResult expireDelegationToken(byte[] hmac)
      Expire a Delegation Token.

      This is a convenience method for expireDelegationToken(byte[], ExpireDelegationTokenOptions) with default options. This will expire the token immediately. See the overload for more details.

      Parameters:
      hmac - HMAC of the Delegation token
      Returns:
      The ExpireDelegationTokenResult.
    • expireDelegationToken

      ExpireDelegationTokenResult expireDelegationToken(byte[] hmac, ExpireDelegationTokenOptions options)
      Expire a Delegation Token.

      This operation is supported by brokers with version 1.1.0 or higher.

      The following exceptions can be anticipated when calling get() on the futures obtained from the expiryTimestamp() method of the returned ExpireDelegationTokenResult

      Parameters:
      hmac - HMAC of the Delegation token
      options - The options to use when expiring delegation token.
      Returns:
      The ExpireDelegationTokenResult.
    • describeDelegationToken

      default DescribeDelegationTokenResult describeDelegationToken()
      Describe the Delegation Tokens.

      This is a convenience method for describeDelegationToken(DescribeDelegationTokenOptions) with default options. This will return all the user owned tokens and tokens where user have Describe permission. See the overload for more details.

      Returns:
      The DescribeDelegationTokenResult.
    • describeDelegationToken

      Describe the Delegation Tokens.

      This operation is supported by brokers with version 1.1.0 or higher.

      The following exceptions can be anticipated when calling get() on the futures obtained from the delegationTokens() method of the returned DescribeDelegationTokenResult

      Parameters:
      options - The options to use when describing delegation tokens.
      Returns:
      The DescribeDelegationTokenResult.
    • describeConsumerGroups

      DescribeConsumerGroupsResult describeConsumerGroups(Collection<String> groupIds, DescribeConsumerGroupsOptions options)
      Describe some group IDs in the cluster.
      Parameters:
      groupIds - The IDs of the groups to describe.
      options - The options to use when describing the groups.
      Returns:
      The DescribeConsumerGroupResult.
    • describeConsumerGroups

      default DescribeConsumerGroupsResult describeConsumerGroups(Collection<String> groupIds)
      Describe some group IDs in the cluster, with the default options.

      This is a convenience method for describeConsumerGroups(Collection, DescribeConsumerGroupsOptions) with default options. See the overload for more details.

      Parameters:
      groupIds - The IDs of the groups to describe.
      Returns:
      The DescribeConsumerGroupResult.
    • listConsumerGroups

      List the consumer groups available in the cluster.
      Parameters:
      options - The options to use when listing the consumer groups.
      Returns:
      The ListGroupsResult.
    • listConsumerGroups

      default ListConsumerGroupsResult listConsumerGroups()
      List the consumer groups available in the cluster with the default options.

      This is a convenience method for listConsumerGroups(ListConsumerGroupsOptions) with default options. See the overload for more details.

      Returns:
      The ListGroupsResult.
    • listConsumerGroupOffsets

      ListConsumerGroupOffsetsResult listConsumerGroupOffsets(String groupId, ListConsumerGroupOffsetsOptions options)
      List the consumer group offsets available in the cluster.
      Parameters:
      options - The options to use when listing the consumer group offsets.
      Returns:
      The ListGroupOffsetsResult
    • listConsumerGroupOffsets

      default ListConsumerGroupOffsetsResult listConsumerGroupOffsets(String groupId)
      List the consumer group offsets available in the cluster with the default options.

      This is a convenience method for listConsumerGroupOffsets(String, ListConsumerGroupOffsetsOptions) with default options.

      Returns:
      The ListGroupOffsetsResult.
    • deleteConsumerGroups

      DeleteConsumerGroupsResult deleteConsumerGroups(Collection<String> groupIds, DeleteConsumerGroupsOptions options)
      Delete consumer groups from the cluster.
      Parameters:
      options - The options to use when deleting a consumer group.
      Returns:
      The DeletConsumerGroupResult.
    • deleteConsumerGroups

      default DeleteConsumerGroupsResult deleteConsumerGroups(Collection<String> groupIds)
      Delete consumer groups from the cluster with the default options.
      Returns:
      The DeleteConsumerGroupResult.
    • deleteConsumerGroupOffsets

      DeleteConsumerGroupOffsetsResult deleteConsumerGroupOffsets(String groupId, Set<TopicPartition> partitions, DeleteConsumerGroupOffsetsOptions options)
      Delete committed offsets for a set of partitions in a consumer group. This will succeed at the partition level only if the group is not actively subscribed to the corresponding topic.
      Parameters:
      options - The options to use when deleting offsets in a consumer group.
      Returns:
      The DeleteConsumerGroupOffsetsResult.
    • deleteConsumerGroupOffsets

      default DeleteConsumerGroupOffsetsResult deleteConsumerGroupOffsets(String groupId, Set<TopicPartition> partitions)
      Delete committed offsets for a set of partitions in a consumer group with the default options. This will succeed at the partition level only if the group is not actively subscribed to the corresponding topic.
      Returns:
      The DeleteConsumerGroupOffsetsResult.
    • electLeaders

      default ElectLeadersResult electLeaders(ElectionType electionType, Set<TopicPartition> partitions)
      Elect a replica as leader for topic partitions.

      This is a convenience method for electLeaders(ElectionType, Set, ElectLeadersOptions) with default options.

      Parameters:
      electionType - The type of election to conduct.
      partitions - The topics and partitions for which to conduct elections.
      Returns:
      The ElectLeadersResult.
    • electLeaders

      ElectLeadersResult electLeaders(ElectionType electionType, Set<TopicPartition> partitions, ElectLeadersOptions options)
      Elect a replica as leader for the given partitions, or for all partitions if the argument to partitions is null.

      This operation is not transactional so it may succeed for some partitions while fail for others.

      It may take several seconds after this method returns success for all the brokers in the cluster to become aware that the partitions have new leaders. During this time, describeTopics(Collection) may not return information about the partitions' new leaders.

      This operation is supported by brokers with version 2.2.0 or later if preferred election is use; otherwise the brokers most be 2.4.0 or higher.

      The following exceptions can be anticipated when calling get() on the future obtained from the returned ElectLeadersResult:

      Parameters:
      electionType - The type of election to conduct.
      partitions - The topics and partitions for which to conduct elections.
      options - The options to use when electing the leaders.
      Returns:
      The ElectLeadersResult.
    • alterPartitionReassignments

      default AlterPartitionReassignmentsResult alterPartitionReassignments(Map<TopicPartition,Optional<NewPartitionReassignment>> reassignments)
      Change the reassignments for one or more partitions. Providing an empty Optional (e.g via Optional.empty()) will revert the reassignment for the associated partition. This is a convenience method for alterPartitionReassignments(Map, AlterPartitionReassignmentsOptions) with default options. See the overload for more details.
    • alterPartitionReassignments

      Change the reassignments for one or more partitions. Providing an empty Optional (e.g via Optional.empty()) will revert the reassignment for the associated partition.

      The following exceptions can be anticipated when calling get() on the futures obtained from the returned AlterPartitionReassignmentsResult:

      Parameters:
      reassignments - The reassignments to add, modify, or remove. See NewPartitionReassignment.
      options - The options to use.
      Returns:
      The result.
    • listPartitionReassignments

      default ListPartitionReassignmentsResult listPartitionReassignments()
      List all of the current partition reassignments This is a convenience method for listPartitionReassignments(ListPartitionReassignmentsOptions) with default options. See the overload for more details.
    • listPartitionReassignments

      default ListPartitionReassignmentsResult listPartitionReassignments(Set<TopicPartition> partitions)
      List the current reassignments for the given partitions This is a convenience method for listPartitionReassignments(Set, ListPartitionReassignmentsOptions) with default options. See the overload for more details.
    • listPartitionReassignments

      default ListPartitionReassignmentsResult listPartitionReassignments(Set<TopicPartition> partitions, ListPartitionReassignmentsOptions options)
      List the current reassignments for the given partitions

      The following exceptions can be anticipated when calling get() on the futures obtained from the returned ListPartitionReassignmentsResult:

      Parameters:
      partitions - The topic partitions to list reassignments for.
      options - The options to use.
      Returns:
      The result.
    • listPartitionReassignments

      default ListPartitionReassignmentsResult listPartitionReassignments(ListPartitionReassignmentsOptions options)
      List all of the current partition reassignments

      The following exceptions can be anticipated when calling get() on the futures obtained from the returned ListPartitionReassignmentsResult:

      Parameters:
      options - The options to use.
      Returns:
      The result.
    • listPartitionReassignments

      Parameters:
      partitions - the partitions we want to get reassignment for, or an empty optional if we want to get the reassignments for all partitions in the cluster
      options - The options to use.
      Returns:
      The result.
    • removeMembersFromConsumerGroup

      RemoveMembersFromConsumerGroupResult removeMembersFromConsumerGroup(String groupId, RemoveMembersFromConsumerGroupOptions options)
      Remove members from the consumer group by given member identities.

      For possible error codes, refer to LeaveGroupResponse.

      Parameters:
      groupId - The ID of the group to remove member from.
      options - The options to carry removing members' information.
      Returns:
      The MembershipChangeResult.
    • alterConsumerGroupOffsets

      default AlterConsumerGroupOffsetsResult alterConsumerGroupOffsets(String groupId, Map<TopicPartition,OffsetAndMetadata> offsets)

      Alters offsets for the specified group. In order to succeed, the group must be empty.

      This is a convenience method for alterConsumerGroupOffsets(String, Map, AlterConsumerGroupOffsetsOptions) with default options. See the overload for more details.

      Parameters:
      groupId - The group for which to alter offsets.
      offsets - A map of offsets by partition with associated metadata.
      Returns:
      The AlterOffsetsResult.
    • alterConsumerGroupOffsets

      Alters offsets for the specified group. In order to succeed, the group must be empty.

      This operation is not transactional so it may succeed for some partitions while fail for others.

      Parameters:
      groupId - The group for which to alter offsets.
      offsets - A map of offsets by partition with associated metadata. Partitions not specified in the map are ignored.
      options - The options to use when altering the offsets.
      Returns:
      The AlterOffsetsResult.
    • listOffsets

      default ListOffsetsResult listOffsets(Map<TopicPartition,OffsetSpec> topicPartitionOffsets)

      List offset for the specified partitions and OffsetSpec. This operation enables to find the beginning offset, end offset as well as the offset matching a timestamp in partitions.

      This is a convenience method for listOffsets(Map, ListOffsetsOptions)

      Parameters:
      topicPartitionOffsets - The mapping from partition to the OffsetSpec to look up.
      Returns:
      The ListOffsetsResult.
    • listOffsets

      ListOffsetsResult listOffsets(Map<TopicPartition,OffsetSpec> topicPartitionOffsets, ListOffsetsOptions options)

      List offset for the specified partitions. This operation enables to find the beginning offset, end offset as well as the offset matching a timestamp in partitions.

      Parameters:
      topicPartitionOffsets - The mapping from partition to the OffsetSpec to look up.
      options - The options to use when retrieving the offsets
      Returns:
      The ListOffsetsResult.
    • describeClientQuotas

      default DescribeClientQuotasResult describeClientQuotas(ClientQuotaFilter filter)
      Describes all entities matching the provided filter that have at least one client quota configuration value defined.

      This is a convenience method for describeClientQuotas(ClientQuotaFilter, DescribeClientQuotasOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 2.6.0 or higher.

      Parameters:
      filter - the filter to apply to match entities
      Returns:
      the DescribeClientQuotasResult containing the result
    • describeClientQuotas

      Describes all entities matching the provided filter that have at least one client quota configuration value defined.

      The following exceptions can be anticipated when calling get() on the future from the returned DescribeClientQuotasResult:

      This operation is supported by brokers with version 2.6.0 or higher.

      Parameters:
      filter - the filter to apply to match entities
      options - the options to use
      Returns:
      the DescribeClientQuotasResult containing the result
    • alterClientQuotas

      default AlterClientQuotasResult alterClientQuotas(Collection<ClientQuotaAlteration> entries)
      Alters client quota configurations with the specified alterations.

      This is a convenience method for alterClientQuotas(Collection, AlterClientQuotasOptions) with default options. See the overload for more details.

      This operation is supported by brokers with version 2.6.0 or higher.

      Parameters:
      entries - the alterations to perform
      Returns:
      the AlterClientQuotasResult containing the result
    • alterClientQuotas

      Alters client quota configurations with the specified alterations.

      Alterations for a single entity are atomic, but across entities is not guaranteed. The resulting per-entity error code should be evaluated to resolve the success or failure of all updates.

      The following exceptions can be anticipated when calling get() on the futures obtained from the returned AlterClientQuotasResult:

      • ClusterAuthorizationException If the authenticated user didn't have alter access to the cluster.
      • InvalidRequestException If the request details are invalid. e.g., a configuration key was specified more than once for an entity.
      • TimeoutException If the request timed out before the alterations could finish. It cannot be guaranteed whether the update succeed or not.

      This operation is supported by brokers with version 2.6.0 or higher.

      Parameters:
      entries - the alterations to perform
      Returns:
      the AlterClientQuotasResult containing the result
    • describeUserScramCredentials

      default DescribeUserScramCredentialsResult describeUserScramCredentials()
      Describe all SASL/SCRAM credentials.

      This is a convenience method for describeUserScramCredentials(List, DescribeUserScramCredentialsOptions)

      Returns:
      The DescribeUserScramCredentialsResult.
    • describeUserScramCredentials

      default DescribeUserScramCredentialsResult describeUserScramCredentials(List<String> users)
      Describe SASL/SCRAM credentials for the given users.

      This is a convenience method for describeUserScramCredentials(List, DescribeUserScramCredentialsOptions)

      Parameters:
      users - the users for which credentials are to be described; all users' credentials are described if null or empty.
      Returns:
      The DescribeUserScramCredentialsResult.
    • describeUserScramCredentials

      Describe SASL/SCRAM credentials.

      The following exceptions can be anticipated when calling get() on the futures from the returned DescribeUserScramCredentialsResult:

      This operation is supported by brokers with version 2.7.0 or higher.

      Parameters:
      users - the users for which credentials are to be described; all users' credentials are described if null or empty.
      options - The options to use when describing the credentials
      Returns:
      The DescribeUserScramCredentialsResult.
    • alterUserScramCredentials

      default AlterUserScramCredentialsResult alterUserScramCredentials(List<UserScramCredentialAlteration> alterations)
      Alter SASL/SCRAM credentials for the given users.

      This is a convenience method for alterUserScramCredentials(List, AlterUserScramCredentialsOptions)

      Parameters:
      alterations - the alterations to be applied
      Returns:
      The AlterUserScramCredentialsResult.
    • alterUserScramCredentials

      Alter SASL/SCRAM credentials.

      The following exceptions can be anticipated when calling get() any of the futures from the returned AlterUserScramCredentialsResult:

      This operation is supported by brokers with version 2.7.0 or higher.

      Parameters:
      alterations - the alterations to be applied
      options - The options to use when altering the credentials
      Returns:
      The AlterUserScramCredentialsResult.
    • describeFeatures

      default DescribeFeaturesResult describeFeatures()
      Describes finalized as well as supported features.

      This is a convenience method for describeFeatures(DescribeFeaturesOptions) with default options. See the overload for more details.

      Returns:
      the DescribeFeaturesResult containing the result
    • describeFeatures

      Describes finalized as well as supported features. The request is issued to any random broker.

      The following exceptions can be anticipated when calling get() on the future from the returned DescribeFeaturesResult:

      • TimeoutException If the request timed out before the describe operation could finish.

      Parameters:
      options - the options to use
      Returns:
      the DescribeFeaturesResult containing the result
    • updateFeatures

      UpdateFeaturesResult updateFeatures(Map<String,FeatureUpdate> featureUpdates, UpdateFeaturesOptions options)
      Applies specified updates to finalized features. This operation is not transactional so some updates may succeed while the rest may fail.

      The API takes in a map of finalized feature names to FeatureUpdate that needs to be applied. Each entry in the map specifies the finalized feature to be added or updated or deleted, along with the new max feature version level value. This request is issued only to the controller since the API is only served by the controller. The return value contains an error code for each supplied FeatureUpdate, and the code indicates if the update succeeded or failed in the controller.

      • Downgrade of feature version level is not a regular operation/intent. It is only allowed in the controller if the FeatureUpdate has the allowDowngrade flag set. Setting this flag conveys user intent to attempt downgrade of a feature max version level. Note that despite the allowDowngrade flag being set, certain downgrades may be rejected by the controller if it is deemed impossible.
      • Deletion of a finalized feature version is not a regular operation/intent. It could be done by setting the allowDowngrade flag to true in the FeatureUpdate, and, setting the max version level to a value less than 1.

      The following exceptions can be anticipated when calling get() on the futures obtained from the returned UpdateFeaturesResult:

      This operation is supported by brokers with version 2.7.0 or higher.

      Parameters:
      featureUpdates - the map of finalized feature name to FeatureUpdate
      options - the options to use
      Returns:
      the UpdateFeaturesResult containing the result
    • unregisterBroker

      @Unstable default UnregisterBrokerResult unregisterBroker(int brokerId)
      Unregister a broker.

      This operation does not have any effect on partition assignments. It is supported only on Kafka clusters which use Raft to store metadata, rather than ZooKeeper. This is a convenience method for unregisterBroker(int, UnregisterBrokerOptions)

      Parameters:
      brokerId - the broker id to unregister.
      Returns:
      the UnregisterBrokerResult containing the result
    • unregisterBroker

      @Unstable UnregisterBrokerResult unregisterBroker(int brokerId, UnregisterBrokerOptions options)
      Unregister a broker.

      This operation does not have any effect on partition assignments. It is supported only on Kafka clusters which use Raft to store metadata, rather than ZooKeeper. The following exceptions can be anticipated when calling get() on the future from the returned UnregisterBrokerResult:

      • TimeoutException If the request timed out before the describe operation could finish.
      • UnsupportedVersionException If the software is too old to support the unregistration API, or if the cluster is not using Raft to store metadata.

      Parameters:
      brokerId - the broker id to unregister.
      options - the options to use.
      Returns:
      the UnregisterBrokerResult containing the result
    • describeProducers

      default DescribeProducersResult describeProducers(Collection<TopicPartition> partitions)
      Describe producer state on a set of topic partitions. See describeProducers(Collection, DescribeProducersOptions) for more details.
      Parameters:
      partitions - The set of partitions to query
      Returns:
      The result
    • describeProducers

      Describe active producer state on a set of topic partitions. Unless a specific broker is requested through DescribeProducersOptions.brokerId(int), this will query the partition leader to find the producer state.
      Parameters:
      partitions - The set of partitions to query
      options - Options to control the method behavior
      Returns:
      The result
    • describeTransactions

      default DescribeTransactionsResult describeTransactions(Collection<String> transactionalIds)
      Describe the state of a set of transactional IDs. See describeTransactions(Collection, DescribeTransactionsOptions) for more details.
      Parameters:
      transactionalIds - The set of transactional IDs to query
      Returns:
      The result
    • describeTransactions

      DescribeTransactionsResult describeTransactions(Collection<String> transactionalIds, DescribeTransactionsOptions options)
      Describe the state of a set of transactional IDs from the respective transaction coordinators, which are dynamically discovered.
      Parameters:
      transactionalIds - The set of transactional IDs to query
      options - Options to control the method behavior
      Returns:
      The result
    • abortTransaction

      default AbortTransactionResult abortTransaction(AbortTransactionSpec spec)
      Forcefully abort a transaction which is open on a topic partition. See abortTransaction(AbortTransactionSpec, AbortTransactionOptions) for more details.
      Parameters:
      spec - The transaction specification including topic partition and producer details
      Returns:
      The result
    • abortTransaction

      Forcefully abort a transaction which is open on a topic partition. This will send a `WriteTxnMarkers` request to the partition leader in order to abort the transaction. This requires administrative privileges.
      Parameters:
      spec - The transaction specification including topic partition and producer details
      options - Options to control the method behavior (including filters)
      Returns:
      The result
    • listTransactions

      default ListTransactionsResult listTransactions()
      List active transactions in the cluster. See listTransactions(ListTransactionsOptions) for more details.
      Returns:
      The result
    • listTransactions

      List active transactions in the cluster. This will query all potential transaction coordinators in the cluster and collect the state of all transactions. Users should typically attempt to reduce the size of the result set using ListTransactionsOptions.filterProducerIds(Collection) or ListTransactionsOptions.filterStates(Collection)
      Parameters:
      options - Options to control the method behavior (including filters)
      Returns:
      The result
    • fenceProducers

      default FenceProducersResult fenceProducers(Collection<String> transactionalIds)
      Fence out all active producers that use any of the provided transactional IDs, with the default options.

      This is a convenience method for fenceProducers(Collection, FenceProducersOptions) with default options. See the overload for more details.

      Parameters:
      transactionalIds - The IDs of the producers to fence.
      Returns:
      The FenceProducersResult.
    • fenceProducers

      FenceProducersResult fenceProducers(Collection<String> transactionalIds, FenceProducersOptions options)
      Fence out all active producers that use any of the provided transactional IDs.
      Parameters:
      transactionalIds - The IDs of the producers to fence.
      options - The options to use when fencing the producers.
      Returns:
      The FenceProducersResult.
    • metrics

      Map<MetricName,? extends Metric> metrics()
      Get the metrics kept by the adminClient