-
Notifications
You must be signed in to change notification settings - Fork 14.6k
KAFKA-18904: kafka-configs.sh return resource doesn't exist message [3/N] #19808
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Changes from 4 commits
a20928d
939eac8
2161248
d87c770
9858d88
298f347
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change | ||
---|---|---|---|---|
|
@@ -6,7 +6,7 @@ | |||
* (the "License"); you may not use this file except in compliance with | ||||
* the License. You may obtain a copy of the License at | ||||
* | ||||
* http://www.apache.org/licenses/LICENSE-2.0 | ||||
* http://www.apache.org/licenses/LICENSE-2.0 | ||||
* | ||||
* Unless required by applicable law or agreed to in writing, software | ||||
* distributed under the License is distributed on an "AS IS" BASIS, | ||||
|
@@ -21,7 +21,7 @@ import joptsimple._ | |||
import kafka.server.DynamicConfig | ||||
import kafka.utils.Implicits._ | ||||
import kafka.utils.Logging | ||||
import org.apache.kafka.clients.admin.{Admin, AlterClientQuotasOptions, AlterConfigOp, AlterConfigsOptions, ConfigEntry, DescribeClusterOptions, DescribeConfigsOptions, ListTopicsOptions, ScramCredentialInfo, UserScramCredentialDeletion, UserScramCredentialUpsertion, ScramMechanism => PublicScramMechanism} | ||||
import org.apache.kafka.clients.admin.{Admin, AlterClientQuotasOptions, AlterConfigOp, AlterConfigsOptions, ConfigEntry, DescribeClusterOptions, DescribeConfigsOptions, ListConfigResourcesOptions, ListTopicsOptions, ScramCredentialInfo, UserScramCredentialDeletion, UserScramCredentialUpsertion, ScramMechanism => PublicScramMechanism} | ||||
import org.apache.kafka.common.config.ConfigResource | ||||
import org.apache.kafka.common.errors.{InvalidConfigurationException, UnsupportedVersionException} | ||||
import org.apache.kafka.common.internals.Topic | ||||
|
@@ -342,6 +342,42 @@ object ConfigCommand extends Logging { | |||
} | ||||
|
||||
private def describeResourceConfig(adminClient: Admin, entityType: String, entityName: Option[String], describeAll: Boolean): Unit = { | ||||
if (!describeAll) { | ||||
entityName.foreach { name => | ||||
entityType match { | ||||
case TopicType => | ||||
Topic.validate(name) | ||||
if (!adminClient.listTopics(new ListTopicsOptions().listInternal(true)).names.get.contains(name)) { | ||||
System.out.println(s"The $entityType '$name' doesn't exist and doesn't have dynamic config.") | ||||
return | ||||
} | ||||
case BrokerType | BrokerLoggerConfigType => | ||||
if (adminClient.describeCluster.nodes.get.stream.anyMatch(_.idString == name)) { | ||||
// valid broker id | ||||
} else if (name == BrokerDefaultEntityName) { | ||||
// default broker configs | ||||
} else { | ||||
System.out.println(s"The $entityType '$name' doesn't exist and doesn't have dynamic config.") | ||||
return | ||||
} | ||||
case ClientMetricsType => | ||||
if (adminClient.listConfigResources(java.util.Set.of(ConfigResource.Type.CLIENT_METRICS), new ListConfigResourcesOptions).all.get | ||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. nit: Can't this just be util.Set.of? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I prefer to leave this as
We can do some refactor for Scala code in core module after this PR. Or we can refactor it when migrating to Java eventually. |
||||
.stream.noneMatch(_.name == name)) { | ||||
System.out.println(s"The $entityType '$name' doesn't exist and doesn't have dynamic config.") | ||||
return | ||||
} | ||||
case GroupType => | ||||
if (adminClient.listGroups().all.get.stream.noneMatch(_.groupId() == name) && | ||||
adminClient.listConfigResources(java.util.Set.of(ConfigResource.Type.GROUP), new ListConfigResourcesOptions).all.get | ||||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. ditto |
||||
.stream.noneMatch(_.name == name)) { | ||||
System.out.println(s"The $entityType '$name' doesn't exist and doesn't have dynamic config.") | ||||
return | ||||
} | ||||
case entityType => throw new IllegalArgumentException(s"Invalid entity type: $entityType") | ||||
} | ||||
} | ||||
} | ||||
|
||||
val entities = entityName | ||||
.map(name => List(name)) | ||||
.getOrElse(entityType match { | ||||
|
@@ -350,9 +386,10 @@ object ConfigCommand extends Logging { | |||
case BrokerType | BrokerLoggerConfigType => | ||||
adminClient.describeCluster(new DescribeClusterOptions()).nodes().get().asScala.map(_.idString).toSeq :+ BrokerDefaultEntityName | ||||
case ClientMetricsType => | ||||
adminClient.listClientMetricsResources().all().get().asScala.map(_.name).toSeq | ||||
adminClient.listConfigResources(java.util.Set.of(ConfigResource.Type.CLIENT_METRICS), new ListConfigResourcesOptions).all().get().asScala.map(_.name).toSeq | ||||
case GroupType => | ||||
adminClient.listGroups().all.get.asScala.map(_.groupId).toSeq | ||||
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 | ||||
case entityType => throw new IllegalArgumentException(s"Invalid entity type: $entityType") | ||||
}) | ||||
|
||||
|
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -156,6 +156,12 @@ public void describeClientMetrics(ClientMetricsCommandOptions opts) throws Excep | |
|
||
List<String> entities; | ||
if (entityNameOpt.isPresent()) { | ||
if (adminClient.listConfigResources(Set.of(ConfigResource.Type.CLIENT_METRICS), new ListConfigResourcesOptions()) | ||
.all().get(30, TimeUnit.SECONDS).stream() | ||
.noneMatch(resource -> resource.name().equals(entityNameOpt.get()))) { | ||
System.out.println("The client-metric " + entityNameOpt.get() + " doesn't exist and doesn't have dynamic config."); | ||
|
||
return; | ||
} | ||
entities = Collections.singletonList(entityNameOpt.get()); | ||
} else { | ||
Collection<ConfigResource> resources = adminClient | ||
|
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
The variable
entityType
contains the pluralized noun such as topics or groups. As a result, the message is not grammatical, for exampleThe groups 'G2' doesn't exist and doesn't have dynamic config.
. This problem is evident for all entity types.Probably easiest here is just to use a slightly different string for each type such as
s"The topic '$name' doesn't exist and doesn't have dynamic config."
.There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Alternatively, and this is what they do elsewhere in this tool, use
${entityType.dropRight(1)}
.