Skip to content

KAFKA-20111: Handle pre-4.1 brokers in kafka-configs.sh for groups#21385

Open
AndrewJSchofield wants to merge 5 commits intoapache:trunkfrom
AndrewJSchofield:KAFKA-20111
Open

KAFKA-20111: Handle pre-4.1 brokers in kafka-configs.sh for groups#21385
AndrewJSchofield wants to merge 5 commits intoapache:trunkfrom
AndrewJSchofield:KAFKA-20111

Conversation

@AndrewJSchofield
Copy link
Member

@AndrewJSchofield AndrewJSchofield commented Feb 2, 2026

KIP-1142 introduced Admin.listConfigResources() for listing resources
which have configurations, but which may not actually exist as run-time
entities such as consumer groups which have not yet had any members.
Internally, this works by using v1 of the LIST_CONFIG_RESOURCES RPC
which is only supported in AK 4.1 and later. As a result, if you try to
describe the config for groups with older brokers, they do not support
v1 of the RPC and fail with UnsupportedVersionException. This PR
handles the exception in the config tool since the exception is harmless
and gives the same behaviour as we used to get with earlier versions.

@github-actions github-actions bot added core Kafka Broker small Small PRs labels Feb 2, 2026
@github-actions github-actions bot added the tools label Feb 2, 2026
@AndrewJSchofield AndrewJSchofield marked this pull request as draft February 2, 2026 21:59
@AndrewJSchofield AndrewJSchofield marked this pull request as ready for review February 3, 2026 20:34
Copy link
Member

@chia7712 chia7712 left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@AndrewJSchofield thanks for this fix

case GroupType =>
adminClient.listGroups().all.get.asScala.map(_.groupId).toSet ++
adminClient.listConfigResources(java.util.Set.of(ConfigResource.Type.GROUP), new ListConfigResourcesOptions).all().get().asScala.map(_.name).toSet
adminClient.listGroups().all.get.asScala.map(_.groupId).toSet ++ listGroupConfigResources(adminClient).map(resources => resources.asScala.map(_.name).toSet).getOrElse(Set() ++ entityName)
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

entityName is None, so Set() ++ entityName could be replaced by Set.empty

}

private def listGroupConfigResources(adminClient: Admin): Option[java.util.Collection[ConfigResource]] = {
try {
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

    try {
      Some(adminClient.listConfigResources(java.util.Set.of(ConfigResource.Type.GROUP), new ListConfigResourcesOptions).all.get)
    } catch {
      // (KIP-1142) 4.1+ admin client vs older broker: treat UnsupportedVersion as None
      case e: ExecutionException if e.getCause.isInstanceOf[UnsupportedVersionException] => None
    }

public ListConfigResourcesResult listConfigResources(Set<ConfigResource.Type> configResourceTypes, ListConfigResourcesOptions options) {
ConfigResource.Type type = configResourceTypes.iterator().next();
assertEquals(ConfigResource.Type.GROUP, type);
future.completeExceptionally(new UnsupportedVersionException("The v0 ListConfigResources only supports CLIENT_METRICS"));
Copy link
Member

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Should we add a test case for other exceptions to ensure they are propagated correctly and not swallowed

Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

core Kafka Broker small Small PRs tools

Projects

None yet

Development

Successfully merging this pull request may close these issues.

2 participants