Commit Graph

10157 Commits

Author SHA1 Message Date
Chris Egerton ec22af94a6
KAFKA-13613: Remove hard dependency on HmacSHA256 algorithm for Connect (#11894)
Reviewers: Mickael Maison <mickael.maison@gmail.com>

, Tom Bentley <tbentley@redhat.com>
2022-07-05 12:34:23 +02:00
Thomas Cooper aa735062eb
Upgrade Netty and Jackson versions for CVE fixes [KAFKA-14044] (#12376)
Reviewers: Luke Chen <showuon@gmail.com>
2022-07-05 14:16:18 +08:00
Matthew de Detrich 4e6326f889
KAFKA-13957: Fix flaky shouldQuerySpecificActivePartitionStores test (#12289)
Currently the tests fail because there is a missing predicate in the retrievableException which causes the test to fail, i.e. the current predicates

containsString("Cannot get state store source-table because the stream thread is PARTITIONS_ASSIGNED, not RUNNING"),
containsString("The state store, source-table, may have migrated to another instance"),
containsString("Cannot get state store source-table because the stream thread is STARTING, not RUNNING")

wasn't complete. Another one needed to be added, namely "The specified partition 1 for store source-table does not exist.". This is because its possible for

assertThat(getStore(kafkaStreams2, storeQueryParam2).get(key), is(nullValue()));

or

assertThat(getStore(kafkaStreams1, storeQueryParam2).get(key), is(nullValue()));

(depending on which branch) to be thrown, i.e. see

org.apache.kafka.streams.errors.InvalidStateStorePartitionException: The specified partition 1 for store source-table does not exist.

	at org.apache.kafka.streams.state.internals.WrappingStoreProvider.stores(WrappingStoreProvider.java:63)
	at org.apache.kafka.streams.state.internals.CompositeReadOnlyKeyValueStore.get(CompositeReadOnlyKeyValueStore.java:53)
	at org.apache.kafka.streams.integration.StoreQueryIntegrationTest.lambda$shouldQuerySpecificActivePartitionStores$5(StoreQueryIntegrationTest.java:223)
	at org.apache.kafka.streams.integration.StoreQueryIntegrationTest.retryUntil(StoreQueryIntegrationTest.java:579)
	at org.apache.kafka.streams.integration.StoreQueryIntegrationTest.shouldQuerySpecificActivePartitionStores(StoreQueryIntegrationTest.java:186)

This happens when the stream hasn't been initialized yet. I have run the test around 12k times using Intellij's JUnit testing framework without any flaky failures. The PR also does some minor refactoring regarding moving the list of predicates into their own functions.

Co-authored-by: Bruno Cadonna <cadonna@apache.org>

Reviewer: Bruno Cadonna <cadonna@apache.org>
2022-07-04 20:26:53 +02:00
Guozhang Wang ae570f5953
HOTFIX: Correct ordering of input buffer and enforced processing sensors (#12363)
1. As titled, fix the right constructor param ordering.
2. Also added a few more loglines.

Reviewers: Matthias J. Sax <matthias@confluent.io>, Sagar Rao <sagarmeansocean@gmail.com>, Hao Li <1127478+lihaosky@users.noreply.github.com>
2022-07-03 10:02:59 -07:00
Bruno Cadonna a82a8e02ce
MINOR: Fix static mock usage in TaskMetricsTest (#12373)
Before this PR the calls to the static methods on StreamsMetricsImpl were just calls and not a verification on the mock. This miss happened during the switch from EasyMock to Mockito.

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2022-07-02 18:48:07 -07:00
RivenSun bad475166f
MINOR: Add indent space after hyperlink in `docs/upgrade.html` (#12353)
Reviewers: Kvicii <42023367+Kvicii@users.noreply.github.com>, Divij Vaidya <divijvaidya13@gmail.com>, Jason Gustafson <jason@confluent.io>
2022-07-01 10:30:32 -07:00
Nikolay c12348ac98
MINOR: Remove unused code from BrokerEndPoint (#12368)
Removes unused methods from `BrokerEndPoint`:

* `createBrokerEndPoint(Int, String)`
* `readFrom(buffer: ByteBuffer)`
* `connectionString(): String`
* `writeTo(buffer: ByteBuffer)`
* `sizeInBytes: Int`

Reviewers: dengziming <dengziming1993@gmail.com>, Luke Chen <showuon@gmail.com>, Jason Gustafson <jason@confluent.io>
2022-07-01 10:28:37 -07:00
Jason Gustafson 6bf5bfc298
KAFKA-14036; Set local time in `ControllerApis` when `handle` returns (#12372)
In `ControllerApis`, we are missing the logic to set the local processing end time after `handle` returns. As a consequence of this, the remote time ends up reported as the local time in the request level metrics. The patch adds the same logic we have in `KafkaApis` to set `apiLocalCompleteTimeNanos`.

Reviewers: José Armando García Sancio <jsancio@gmail.com>
2022-06-30 21:07:21 -07:00
Niket c19398ee66
KAFKA-14035; Fix NPE in `SnapshottableHashTable::mergeFrom()` (#12371)
The NPE causes the kraft controller to be in an inconsistent state. 

Reviewers: Jason Gustafson <jason@confluent.io>
2022-06-30 21:03:54 -07:00
Prashanth Joseph Babu 1daa149730
MINOR: record lag max metric documentation enhancement (#12367)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2022-06-30 09:30:14 -07:00
Guozhang Wang 3faa6cf6d0
MINOR: Use mock time in DefaultStateUpdaterTest (#12344)
For most tests we would need an auto-ticking mock timer to work with draining-with-timeout functions.
For tests that check for never checkpoint we need no auto-ticking timer to control exactly how much time elapsed.

Reviewers: Bruno Cadonna <cadonna@apache.org>
2022-06-29 12:33:00 -07:00
Guozhang Wang ababc4261b
[9/N][Emit final] Emit final for session window aggregations (#12204)
* Add a new API for session windows to range query session window by end time (KIP related).
* Augment session window aggregator with emit strategy.
* Minor: consolidated some dup classes.
* Test: unit test on session window aggregator.

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2022-06-29 09:22:37 -07:00
Luke Chen 08e509459b
KAFKA-14010: AlterPartition request won't retry when receiving retriable error (#12329)
When submitting the AlterIsr request, we register a future listener to handle the response. When receiving retriable error, we expected the AlterIsr request will get retried. And then, we'll re-submit the request again.

However, before the future listener got called, we didn't clear the `unsentIsrUpdates`, which causes we failed to "enqueue" the request because we thought there's an in-flight request. We use "try/finally" to make sure the unsentIsrUpdates got cleared, but it happened "after" we retry the request	

Reviewers: David Jacot <djacot@confluent.io>, dengziming <dengziming1993@gmail.com>
2022-06-29 20:53:36 +08:00
CHUN-HAO TANG 6ac7f4ea8f
KAFKA-13821: Update Kafka Streams WordCount demo to new Processor API (#12139)
https://issues.apache.org/jira/browse/KAFKA-13821

Reviewers: Jorge Esteban Quilcate Otoya <quilcate.jorge@gmail.com>, Bill Bejeck <bbejeck@apache.org>
2022-06-28 21:39:32 -04:00
Tom Kaszuba 025e47b833
KAFKA-13963: Clarified TopologyDescription JavaDoc for Processors API forward() calls (#12293)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2022-06-27 18:50:34 -07:00
Jason Gustafson d654bc1b15
MINOR: Support KRaft in GroupAuthorizerIntegrationTest (#12336)
Support KRaft in `GroupAuthorizerIntegrationTest`. 

Reviewers: David Arthur <mumrah@gmail.com>
2022-06-27 16:01:15 -07:00
Alyssa Huang a3c7017ff7
MINOR: Support KRaft in ReplicaFetchTest (#12345)
Reviewers: Kvicii <42023367+Kvicii@users.noreply.github.com>, Jason Gustafson <jason@confluent.io>
2022-06-25 10:25:55 -07:00
Bruno Cadonna 1ceaf30039
KAFKA-10199: Expose tasks in state updater (#12312)
This PR exposes the tasks managed by the state updater. The state updater manages all tasks that were added to the state updater and that have not yet been removed from it by draining one of the output queues.

Reviewers: Guozhang Wang <wangguoz@gmail.com>
2022-06-24 09:33:24 -07:00
Bruno Cadonna 08e27914cc
HOTFIX: Fix NPE in StreamTask#shouldCheckpointState (#12341)
The mocks were not setup correctly in StreamTask#shouldCheckpointState
which caused a null pointer exception during test execution.
2022-06-24 12:19:22 +02:00
David Jacot cf9c3a2eca
MINOR: Fix group coordinator is unavailable log (#12335)
Reviewers: Kvicii <42023367+Kvicii@users.noreply.github.com>, Luke Chen <showuon@gmail.com>
2022-06-24 11:48:09 +02:00
Guozhang Wang 925c628173
KAFKA-10199: Commit the restoration progress within StateUpdater (#12279)
During restoring, we should always commit a.k.a. write checkpoint file regardless of EOS or ALOS, since if there's a failure we would just over-restore them upon recovery so no EOS violations happened.

Also when we complete restore or remove task, we should enforce a checkpoint as well; for failing cases though, we should not write a new one.

Reviewers: Bruno Cadonna <cadonna@apache.org>
2022-06-23 10:46:14 -07:00
David Arthur c6c9da02a8
KAFKA-13966 Prepend bootstrap metadata to controller queue (#12269)
Also fixes flaky QuorumControllerTest#testInvalidBootstrapMetadata

Reviewers: Jason Gustafson <jason@confluent.io>
2022-06-23 11:29:21 -04:00
Chris Egerton d00b7875e0
KAFKA-13987: Isolate REST request timeout changes in Connect integration tests (#12291)
This causes the artificial reductions in the Connect REST request timeout to be more isolated. Specifically, they now only take place in the tests that need them (instead of any tests that happen to be running after the reduction has taken place and before it has been reset), and they are only performed for the requests that are expected to time out, before being immediately reset. This should help reduce spurious test failures (especially in slow environments like Jenkins) for all Connect integration tests that interact with the REST API, not just the BlockingConnectorTest test suite.

Reviewers: Bruno Cadonna <cadonna@apache.org>
2022-06-23 16:56:53 +02:00
Bruno Cadonna 8026a0edd8
MINOR: Fix static mock usage in NamedCacheMetricsTest (#12322)
Before this PR the call to `StreamsMetricsImpl.addAvgAndMinAndMaxToSensor()`
was just a call and not a verification on the mock. This miss happened
during the switch from EasyMock to Mockito.

Reviewers: John Roesler <vvcephei@apache.org>, Guozhang Wang <wangguoz@gmail.com>
2022-06-23 09:45:46 +02:00
Bruno Cadonna 269277f73b
MINOR: Fix static mock usage in ProcessorNodeMetricsTest (#12323)
Before this PR the calls to StreamsMetricsImpl.addInvocationRateAndCountToSensor()
were just calls and not a verification on the mock. This miss happened
during the switch from EasyMock to Mockito.

Reviewers: John Roesler <vvcephei@apache.org>, Guozhang Wang <wangguoz@gmail.com>
2022-06-23 09:45:22 +02:00
Bruno Cadonna 6f5843dae6
MINOR: Fix static mock usage in StateStoreMetricsTest (#12325)
Before this PR the calls to the static methods on
StreamsMetricsImpl were just calls and not a verification
on the mock. This miss happened during the switch from
EasyMock to Mockito.

Reviewers: John Roesler <vvcephei@apache.org>, Guozhang Wang <wangguoz@gmail.com>
2022-06-23 09:44:50 +02:00
Jason Gustafson ead6645123
MINOR: Add ineligible replica reason to log message (#12328)
It's useful if the message about ineligible replicas explains the reason the replica is ineligible.

Reviewers: David Jacot <djacot@confluent.io>
2022-06-22 13:27:47 -07:00
David Jacot 3072b3d23e
MINOR: Fix AlterPartitionManager topic id handling in response handler (#12317)
f83d95d9a2 introduced topic ids in the AlterPartitionRequest/Response and we just found a bug in the request handling logic. The issue is the following.

When the `AlterPartitionManager` receives the response, it builds the `partitionResponses` mapping `TopicIdPartition` to its result. `TopicIdPartition` is built from the response. Therefore if version < 2 is used, `TopicIdPartition` will have the `ZERO` topic id. Then the `AlterPartitionManager` iterates over the item sent to find their response. If an item has a topic id in its `TopicIdPartition` and version < 2 was used, it cannot find it because one has it and the other one has not.

This patch fixes the issue by using `TopicPartition` as a key in the `partitionResponses` map. This ensures that the result can be found regardless of the topic id being set or not.

Note that the case where version 2 is used is handled correctly because we already have logic to get back the topic name from the topic id in order to construct the `TopicPartition`.

`testPartialTopicIds` test was supposed to catch this but it didn't due to the ignorable topic id field being present. This patch fixes the test as well.

Reviewers: Kvicii <42023367+Kvicii@users.noreply.github.com>, Jason Gustafson <jason@confluent.io>
2022-06-21 18:24:33 +02:00
Bruno Cadonna 4d53dd9972
KAFKA-13930: Add 3.2.0 Streams upgrade system tests (#12209)
* KAFKA-13930: Add 3.2.0 Streams upgrade system tests

Apache Kafka 3.2.0 was recently released. Now we need
to test upgrades from 3.2 to trunk in our system tests.

Reviewer: Bill Bejeck <bbejeck@apache.org>
2022-06-21 16:33:40 +02:00
A. Sophie Blee-Goldman 0928666987
MINOR: change Streams topic-level metrics tag from 'topic-name' to 'topic' (#12310)
Changes the tag name from topic-name to just topic to conform to the way this tag is named elsewhere (ie in the clients)
Also:
    - fixes a comment about dynamic topic routing
    - fixes some indentation in MockRecordCollector
    - Undoes the changes to KStreamSplitTest.scala and TestTopicsTest which are no longer necessary after this hotfix

Reviewers: Bruno Cadonna <cadonna@apache.org>
2022-06-21 13:10:36 +02:00
Chris Egerton b7b7615db3
KAFKA-10000: Per-connector offsets topics (#11781)
Implements support for per-connector offsets topics as described in KIP-618.

Reviewers: Luke Chen <showuon@gmail.com>, Tom Bentley <tbentley@redhat.com>
2022-06-21 15:58:17 +08:00
Viktor Somogyi-Vass d65d886798
KAFKA-6945: KIP-373, allow users to create delegation token for others (#10738)
Reviewers: Manikumar Reddy <manikumar.reddy@gmail.com>
2022-06-21 12:51:08 +05:30
James Hughes 30216ea1c5
KAFKA-13998: JoinGroupRequestData 'reason' can be too large (#12298)
The `reason` field cannot contain more than 32767 chars. We did not expect to ever reach this but it turns out that it is possible if the the message provided in the `Throwable` somehow contains the entire stack trace. This patch ensure that the reason crafted based on exceptions remain small.

Co-authored-by: David Jacot <djacot@confluent.io>

Reviewers:  Bruno Cadonna <cadonna@apache.org>, A. Sophie Blee-Goldman <ableegoldman@apache.org>, David Jacot <djacot@confluent.io>
2022-06-20 16:47:02 +02:00
RivenSun 35ee09d43f
MINOR: Remove extra commas in upgrade steps documentation (#12311)
Reviewers: Luke Chen <showuon@gmail.com>
2022-06-20 16:57:49 +08:00
Guozhang Wang cfdd567955
KAFKA-13880: Remove DefaultPartitioner from StreamPartitioner (#12304)
There are some considerata embedded in this seemingly straight-forward PR that I'd like to explain here. The StreamPartitioner is used to send records to three types of topics:

1) repartition topics, where key should never be null.
2) changelog topics, where key should never be null.
3) sink topics, where only non-windowed key could be null and windowed key should still never be null.
Also, the StreamPartitioner is used as part of the IQ to determine which host contains a certain key, as determined by the case 2) above.

This PR's main goal is to remove the deprecated producer's default partitioner, while with those things in mind such that:

We want to make sure for not-null keys, the default murmur2 hash behavior of the streams' partitioner stays consistent with producer's new built-in partitioner.
For null-keys (which is only possible for non-window default stream partition, and is never used for IQ), we would fix the issue that we may never rotate to a new partitioner by setting the partition as null hence relying on the newly introduced built-in partitioner.

Reviewers: Artem Livshits <84364232+artemlivshits@users.noreply.github.com>, Matthias J. Sax <matthias@confluent.io>
2022-06-17 20:17:02 -07:00
RivenSun 1e21201ea2
KAFKA-13890: Improve documentation of `ssl.keystore.type` and `ssl.truststore.type` (#12226)
Reviewers: Mickael Maison <mickael.maison@gmail.com>

, David Jacot <djacot@confluent.io>, Kvicii <kvicii.yu@gmail.com>
2022-06-17 16:31:13 +02:00
Divij Vaidya 17637c4ad5
MINOR: Clean up tmp files created by tests (#12233)
There are a bunch of tests which do not clean up after themselves. This leads to
accumulation of files in the tmp directory of the system on which the tests are
running. 

This code change fixes some of the main culprit tests which leak the files in the
temporary directory.

Reviewers: Ismael Juma <ismael@juma.me.uk>, Kvicii <kvicii.yu@gmail.com>
2022-06-16 16:46:07 -07:00
jnewhouse ee565f5f6b
KAFKA-13939: Only track dirty keys if logging is enabled. (#12263)
InMemoryTimeOrderedKeyValueBuffer keeps a Set of keys that have been seen in order to log them for durability. This set is never used nor cleared if logging is not enabled. Having it be populated creates a memory leak. This change stops populating the set if logging is not enabled.

Reviewers: Divij Vaidya <diviv@amazon.com>, Kvicii <42023367+Kvicii@users.noreply.github.com>, Guozhang Wang <wangguoz@gmail.com>
2022-06-16 14:27:38 -07:00
James Hughes 683d0bbc4c
MINOR: Guard against decrementing `totalCommittedSinceLastSummary` during rebalancing. (#12299)
Reviewers: Matthias J. Sax <matthias@confluent.io>
2022-06-16 09:40:08 -07:00
James Hughes 7ed3748a46
KAFKA-13873 Add ability to Pause / Resume KafkaStreams Topologies (#12161)
This PR adds the ability to pause and resume KafkaStreams instances as well as named/modular topologies (KIP-834).

Co-authored-by: Bruno Cadonna <cadonna@apache.org>

Reviewers: Bonnie Varghese <bvarghese@confluent.io>, Walker Carlson <wcarlson@confluent.io>, Guozhang Wang <guozhang@apache.org>, Bruno Cadonna <cadonna@apache.org>
2022-06-16 16:06:02 +02:00
Niket a126e3a622
KAFKA-13888; Addition of Information in DescribeQuorumResponse about Voter Lag (#12206)
This commit adds an Admin API handler for DescribeQuorum Request and also
adds in two new fields LastFetchTimestamp and LastCaughtUpTimestamp to
the DescribeQuorumResponse as described by KIP-836.

This commit does not implement the newly added fields. Those will be
added in a subsequent commit.

Reviewers: dengziming <dengziming1993@gmail.com>, David Jacot <djacot@confluent.io>, Jason Gustafson <jason@confluent.io>
2022-06-15 09:20:15 -07:00
Ron Dagostino b04937dc65
MINOR: Fix force kill of KRaft colocated controllers in system tests (#11238)
I noticed that a system test using a KRaft cluster with 3 brokers but only 1 co-located controller did not force-kill the second and third broker after shutting down the first broker (the one with the controller).  The issue was a floating point rounding error.  This patch adjusts for the rounding error and also makes the logic work for an even number of controllers.  A local run of `tests/kafkatest/sanity_checks/test_bounce.py` succeeded (and I manually increased the cluster size for the 1 co-located controller case and observed the correct kill behavior: the second and third brokers were force-killed as expected).

Reviewers: Luke Chen <showuon@gmail.com>, José Armando García Sancio <jsancio@users.noreply.github.com>, David Jacot <djacot@confluent.io>
2022-06-15 16:45:00 +02:00
Matthias J. Sax 44edad5bb5
MINOR: improve description of `commit.interval.ms` config (#12169)
Reviewers: Luke Chen <showuon@gmail.com>, Kvicii Y <@Kvicii>, Bruno Cadonna <bruno@confluent.io>, A. Sophie Blee-Goldman <sophie@confluent.io>
2022-06-14 22:29:25 -07:00
Guozhang Wang 39a555ba94
KAFKA-13846: Use the new addMetricsIfAbsent API (#12287)
Use the newly added function to replace the old addMetric function that may throw illegal argument exceptions.

Although in some cases concurrency should not be possible they do not necessarily remain always true in the future, so it's better to use the new API just to be less error-prone.

Reviewers: Bruno Cadonna <cadonna@apache.org>
2022-06-14 16:04:26 -07:00
Ismael Juma f421c008aa MINOR: Remove ReplicaManagerTest.initializeLogAndTopicId (#12276)
The workaround is not required with mockito.

Reviewers: José Armando García Sancio <jsancio@users.noreply.github.com>, Divij Vaidya <diviv@amazon.com>, Kvicii <42023367+Kvicii@users.noreply.github.com>
2022-06-14 09:10:18 -07:00
Mickael Maison 4fcfd9ddc4
KAFKA-13958: Expose logdirs total/usable space via Kafka API (KIP-827) (#12248)
This implements KIP-827: https://cwiki.apache.org/confluence/display/KAFKA/KIP-827%3A+Expose+logdirs+total+and+usable+space+via+Kafka+API

Add TotalBytes and UsableBytes to DescribeLogDirsResponse
Add matching getters on LogDirDescription

Reviewers: Tom Bentley <tbentley@redhat.com>, Divij Vaidya<diviv@amazon.com>, Viktor Somogyi-Vass <viktorsomogyi@gmail.com>, Igor Soarez <soarez@apple.com>
2022-06-14 14:20:29 +02:00
David Jacot f83d95d9a2
KAFKA-13916; Fenced replicas should not be allowed to join the ISR in KRaft (KIP-841, Part 2) (#12181)
This path implements [KIP-841](https://cwiki.apache.org/confluence/display/KAFKA/KIP-841%3A+Fenced+replicas+should+not+be+allowed+to+join+the+ISR+in+KRaft). Specifically, it implements the following:
* It introduces INELIGIBLE_REPLICA and NEW_LEADER_ELECTED error codes.
* The KRaft controller validates the new ISR provided in the AlterPartition request and rejects the call if any replica in the new ISR is not eligible to join the the ISR - e.g. when fenced or shutting down. The leader reverts to the last committed ISR when its request is rejected due to this.
* The partition leader also verifies that a replica is eligible before trying to add it back to the ISR. If it is not eligible, the ISR expansion is not triggered at all.
* Updates the AlterPartition API to use topic ids. Updates the AlterPartition manger to handle topic names/ids. Updates the ZK controller and the KRaft controller to handle topic names/ids depending on the version of the request used.

Reviewers: Artem Livshits <84364232+artemlivshits@users.noreply.github.com>, José Armando García Sancio <jsancio@users.noreply.github.com>, Jason Gustafson <jason@confluent.io>
2022-06-14 13:12:45 +02:00
A. Sophie Blee-Goldman 3189a8648f
HOTFIX: null check keys of ProducerRecord when computing sizeInBytes (#12288)
Minor followup to #12235 that adds a null check on the record key in the new ClientUtils#producerRecordSizeInBytes utility method, as there are valid cases in which we might be sending records with null keys to the Producer, such as a simple builder.stream("non-keyed-input-topic").filter(...).to("output-topic")

Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>
2022-06-13 22:27:06 -07:00
David Arthur cc384054c6
KAFKA-13935 Fix static usages of IBP in KRaft mode (#12250)
* Set the minimum supported MetadataVersion to 3.0-IV1
* Remove MetadataVersion.UNINITIALIZED
* Relocate RPC version mapping for fetch protocols into MetadataVersion
* Replace static IBP calls with dynamic calls to MetadataCache

A side effect of removing the UNINITIALIZED metadata version is that the FeatureControlManager and FeatureImage will initialize themselves with the minimum KRaft version (3.0-IV1).

The rationale for setting the minimum version to 3.0-IV1 is so that we can avoid any cases of KRaft mode running with an old log message format (KIP-724 was introduced in 3.0-IV1). As a side-effect of increasing this minimum version, the feature level values decreased by one.

Reviewers: Jason Gustafson <jason@confluent.io>, Jun Rao <junrao@gmail.com>
2022-06-13 14:23:28 -04:00
vamossagar12 5cab11cf52
KAFKA-13846: Adding overloaded metricOrElseCreate method (#12121)
Reviewers: David Jacot <djacot@confluent.io>, Justine Olshan <jolshan@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
2022-06-13 10:36:39 -07:00