Skip to content

KAFKA-8964: Rename tag client-id for thread-level metrics and below#7429

Merged
guozhangwang merged 4 commits intoapache:trunkfrom
cadonna:AK8964-rename_client_id_tag_key
Oct 8, 2019
Merged

KAFKA-8964: Rename tag client-id for thread-level metrics and below#7429
guozhangwang merged 4 commits intoapache:trunkfrom
cadonna:AK8964-rename_client_id_tag_key

Conversation

@cadonna
Copy link
Member

@cadonna cadonna commented Oct 1, 2019

  • Renamed tag client-id to thread-id for thread-level metrics and below
  • Corrected metrics tag keys for state store that had suffix "-id" instead
    of "state-id"

Committer Checklist (excluded from commit message)

  • Verify design and implementation
  • Verify test coverage and CI build status
  • Verify documentation (including upgrade notes)

@cadonna
Copy link
Member Author

cadonna commented Oct 1, 2019

Call for review: @guozhangwang @vvcephei @bbejeck

@cadonna cadonna changed the title KAFKA-8964: Rename tag client-id to thread-id for thread-level metric… KAFKA-8964: Rename tag client-id for thread-level metrics and below Oct 1, 2019
@bbejeck bbejeck added the streams label Oct 2, 2019
Copy link
Member

@bbejeck bbejeck left a comment

Choose a reason for hiding this comment

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

Thanks for the PR @cadonna overall this LGTM, I just have 2 minor comments.

public final Map<String, String> tagMap(final String... tags) {
final Map<String, String> tagMap = new LinkedHashMap<>();
tagMap.put("client-id", threadName);
if (tags != null) {
Copy link
Member

Choose a reason for hiding this comment

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

Nice refactoring to put this code in a separate method!

Comment on lines +267 to +268
tagMap.get(builtInMetricsVersion.equals(StreamsConfig.METRICS_LATEST) ? StreamsMetricsImpl.THREAD_ID_TAG
: StreamsMetricsImpl.THREAD_ID_TAG_0100_TO_23),
Copy link
Member

Choose a reason for hiding this comment

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

Maybe get the value of this expression on the line above. IMHO it will make it easier to see what the assertThat is doing.

Copy link
Member Author

Choose a reason for hiding this comment

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

Done

Comment on lines +320 to +498
tagMap.get(builtInMetricsVersion.equals(StreamsConfig.METRICS_LATEST) ? StreamsMetricsImpl.THREAD_ID_TAG
: StreamsMetricsImpl.THREAD_ID_TAG_0100_TO_23),
Copy link
Member

Choose a reason for hiding this comment

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

Same as above

Copy link
Member Author

Choose a reason for hiding this comment

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

Done

@bbejeck
Copy link
Member

bbejeck commented Oct 2, 2019

Java 11/2.12, Java 11/2.13 and Java 8 all failed. Test results already cleaned up.

retest this please

@bbejeck
Copy link
Member

bbejeck commented Oct 2, 2019

ping either of @guozhangwang or @vvcephei for a second review

Copy link
Contributor

@vvcephei vvcephei left a comment

Choose a reason for hiding this comment

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

Looks good! Thanks @cadonna

Copy link
Contributor

@guozhangwang guozhangwang left a comment

Choose a reason for hiding this comment

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

Minor comments, otherwise LGTM.

final Map<String, String> tagMap = metrics.tagMap("task-id", context.taskId().toString(), PROCESSOR_NODE_ID_TAG, processorNodeName);
final Map<String, String> allTagMap = metrics.tagMap("task-id", context.taskId().toString(), PROCESSOR_NODE_ID_TAG, "all");
final Map<String, String> tagMap = metrics.nodeLevelTagMap(context.taskId().toString(), processorNodeName);
final Map<String, String> allTagMap =
Copy link
Contributor

Choose a reason for hiding this comment

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

This is still a node-level metrics technically speaking, but just with node-tag pointing to all right? In that can should we just call nodeLevelTagMap(context.taskId().toString(), ROLLUP_VALUE)?

Copy link
Member Author

Choose a reason for hiding this comment

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

Good catch! Of course! I even did as you said for the commitOverTask sensor. I totally overlooked this one.

return tagMap;
}

public Map<String, String> taskLevelTagMap(final String taskName, final String... tags) {
Copy link
Contributor

Choose a reason for hiding this comment

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

If the above comment is valid, then we do not need this overload func here.

Copy link
Member Author

Choose a reason for hiding this comment

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

Right! Will delete!

updatedTags[tags.length] = scopeName + "-id";
updatedTags[tags.length + 1] = entityName;
return tagMap(updatedTags);
return threadLevelTagMap(updatedTags);
Copy link
Contributor

Choose a reason for hiding this comment

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

This makes me thinking: should we clarify in our javadoc that user customized metrics would be recorded at the thread-level specifically, with additional scope tag?

Copy link
Member Author

Choose a reason for hiding this comment

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

That would be good idea. Do you want to update the javadocs in KIP-444?

Copy link
Contributor

Choose a reason for hiding this comment

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

SG.

Copy link
Member Author

Choose a reason for hiding this comment

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

Thank you!

expiredRecordSensor,
"stream-" + metricScope + "-metrics",
metrics.tagMap("task-id", taskName, metricScope + "-id", name()),
metrics.storeLevelTagMap(taskName, metricScope, name()),
Copy link
Contributor

Choose a reason for hiding this comment

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

Good catch!

Copy link
Contributor

Choose a reason for hiding this comment

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

Also since the "incorrect" metrics name is out in our previous releases, but fixing it we are effectively breaking user's compatibility. I think this okay since we are fixing a bug, but would still worth clarifying it in the upgrade guide docs.

Copy link
Member Author

Choose a reason for hiding this comment

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

Ack

Copy link
Contributor

@guozhangwang guozhangwang left a comment

Choose a reason for hiding this comment

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

One meta comment: in KIP-444 we are changing the type from stream-metrics to stream-thread-metrics as well, is it going to be included in future PRs? I did not see it in either of the open ones yet.

@bbejeck
Copy link
Member

bbejeck commented Oct 3, 2019

All failed, test results cleaned up already

retest this please

Copy link
Member Author

@cadonna cadonna left a comment

Choose a reason for hiding this comment

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

@guozhangwang Yes, the type is changed from stream-metrics to stream-thread-metrics in the thread-level refactoring thread that I could not open yet since it is based on this one.

final Map<String, String> tagMap = metrics.tagMap("task-id", context.taskId().toString(), PROCESSOR_NODE_ID_TAG, processorNodeName);
final Map<String, String> allTagMap = metrics.tagMap("task-id", context.taskId().toString(), PROCESSOR_NODE_ID_TAG, "all");
final Map<String, String> tagMap = metrics.nodeLevelTagMap(context.taskId().toString(), processorNodeName);
final Map<String, String> allTagMap =
Copy link
Member Author

Choose a reason for hiding this comment

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

Good catch! Of course! I even did as you said for the commitOverTask sensor. I totally overlooked this one.

return tagMap;
}

public Map<String, String> taskLevelTagMap(final String taskName, final String... tags) {
Copy link
Member Author

Choose a reason for hiding this comment

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

Right! Will delete!

updatedTags[tags.length] = scopeName + "-id";
updatedTags[tags.length + 1] = entityName;
return tagMap(updatedTags);
return threadLevelTagMap(updatedTags);
Copy link
Member Author

Choose a reason for hiding this comment

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

That would be good idea. Do you want to update the javadocs in KIP-444?

expiredRecordSensor,
"stream-" + metricScope + "-metrics",
metrics.tagMap("task-id", taskName, metricScope + "-id", name()),
metrics.storeLevelTagMap(taskName, metricScope, name()),
Copy link
Member Author

Choose a reason for hiding this comment

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

Ack

@bbejeck
Copy link
Member

bbejeck commented Oct 4, 2019

Java 11/2.12 failed with org.apache.kafka.connect.integration.ExampleConnectIntegrationTest.testSourceConnector
Java 11/2.13 failed with org.apache.kafka.connect.integration.ExampleConnectIntegrationTest.testSinkConnector
Java 8 failed with
org.apache.kafka.connect.integration.ExampleConnectIntegrationTest.testSourceConnector

retest this please

…s and below

- Renamed tag client-id to thread-id for thread-level metrics and below
- Corrected metrics tag keys for state store that had suffix "-id" instead
  of "state-id"
@cadonna cadonna force-pushed the AK8964-rename_client_id_tag_key branch from adbcf55 to 6acde48 Compare October 8, 2019 12:08
@cadonna
Copy link
Member Author

cadonna commented Oct 8, 2019

In JDK 11/Scala 2.13 the following tests failed:

kafka.admin.ReassignPartitionsClusterTest.shouldTriggerReassignmentWithZnodePrecedenceOnControllerStartup
org.apache.kafka.streams.integration.KTableKTableForeignKeyJoinIntegrationTest.doLeftJoinFromRightThenDeleteRightEntity

Retest this, please

@guozhangwang
Copy link
Contributor

Test failures on kafka.api.SslAdminClientIntegrationTest.testAclOperations and org.apache.kafka.streams.integration.KTableKTableForeignKeyJoinIntegrationTest.doLeftJoinFromRightThenDeleteRightEntity respectively.

@guozhangwang guozhangwang merged commit e3c2148 into apache:trunk Oct 8, 2019
@guozhangwang
Copy link
Contributor

LGTM, merged to trunk (2.5).

soondenana added a commit to soondenana/kafka that referenced this pull request Feb 7, 2020
Revert "KAFKA-8964: Rename tag client-id for thread-level metrics and below (apache#7429)"
ijuma added a commit to ijuma/kafka that referenced this pull request Apr 28, 2020
…t-for-generated-requests

* apache-github/trunk:
  KAFKA-8932; Add tag for CreateTopicsResponse.TopicConfigErrorCode (KIP-525) (apache#7464)
  KAFKA-8944: Fixed KTable compiler warning. (apache#7393)
  KAFKA-8964: Rename tag client-id for thread-level metrics and below (apache#7429)
  MINOR: remove unused imports in Streams system tests (apache#7468)
  KAFKA-7190; Retain producer state until transactionalIdExpiration time passes (apache#7388)
  KAFKA-8983; AdminClient deleteRecords should not fail all partitions unnecessarily (apache#7449)
  MINOR: Modified Exception handling for KIP-470 (apache#7461)
  KAFKA-7245: Deprecate WindowStore#put(key, value) (apache#7105)
  KAFKA-8179: Part 7, cooperative rebalancing in Streams (apache#7386)
  KAFKA-8985; Add flexible version support to inter-broker APIs (apache#7453)
  MINOR: Bump version to 2.5.0-SNAPSHOT (apache#7455)
@mjsax mjsax added the kip Requires or implements a KIP label Jun 12, 2020
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

kip Requires or implements a KIP streams

Projects

None yet

Development

Successfully merging this pull request may close these issues.

5 participants