Skip to content

Commit

Permalink
KAFKA-14978 ExactlyOnceWorkerSourceTask should remove parent metrics (a…
Browse files Browse the repository at this point in the history
…pache#13690)

Reviewers: Chris Egerton <chrise@aiven.io>, Viktor Somogyi-Vass <viktorsomogyi@gmail.com>
  • Loading branch information
urbandan authored and viktorsomogyi committed May 10, 2023
1 parent 721a917 commit 55ef8a1
Show file tree
Hide file tree
Showing 2 changed files with 21 additions and 0 deletions.
Expand Up @@ -233,6 +233,7 @@ protected void finalOffsetCommit(boolean failed) {
@Override
public void removeMetrics() {
Utils.closeQuietly(transactionMetrics, "source task transaction metrics tracker");
super.removeMetrics();
}

@Override
Expand Down
Expand Up @@ -19,6 +19,7 @@
import org.apache.kafka.clients.producer.Producer;
import org.apache.kafka.clients.producer.RecordMetadata;
import org.apache.kafka.common.KafkaException;
import org.apache.kafka.common.MetricName;
import org.apache.kafka.common.TopicPartition;
import org.apache.kafka.common.errors.InvalidTopicException;
import org.apache.kafka.common.errors.RecordTooLargeException;
Expand Down Expand Up @@ -82,7 +83,9 @@
import java.util.concurrent.TimeoutException;
import java.util.concurrent.atomic.AtomicReference;
import java.util.function.Consumer;
import java.util.stream.Collectors;

import static java.util.Collections.emptySet;
import static org.apache.kafka.connect.integration.MonitorableSourceConnector.TOPIC_CONFIG;
import static org.apache.kafka.connect.runtime.ConnectorConfig.CONNECTOR_CLASS_CONFIG;
import static org.apache.kafka.connect.runtime.ConnectorConfig.KEY_CONVERTER_CLASS_CONFIG;
Expand Down Expand Up @@ -291,6 +294,23 @@ private void createWorkerTask(TargetState initialState, Converter keyConverter,
sourceConfig, Runnable::run, preProducerCheck, postProducerCheck);
}

@Test
public void testRemoveMetrics() {
createWorkerTask();

workerTask.removeMetrics();

assertEquals(emptySet(), filterToTaskMetrics(metrics.metrics().metrics().keySet()));
}

private Set<MetricName> filterToTaskMetrics(Set<MetricName> metricNames) {
return metricNames
.stream()
.filter(m -> metrics.registry().taskGroupName().equals(m.group())
|| metrics.registry().sourceTaskGroupName().equals(m.group()))
.collect(Collectors.toSet());
}

@Test
public void testStartPaused() throws Exception {
createWorkerTask(TargetState.PAUSED);
Expand Down

0 comments on commit 55ef8a1

Please sign in to comment.