-
Notifications
You must be signed in to change notification settings - Fork 14.1k
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
KAFKA-9422: Track the set of topics a connector is using (KIP-558) (#…
…8017) This feature corresponds to KIP-558 and extends how the internal status topic (set via `status.storage.topic` distributed worker config) is used to include information that allows Kafka Connect to keep track which topics a connector is using. The set of topics a connector is actively using, is exposed via a new endpoint that is added to the REST API of Connect workers. * A `GET /connectors/{name}/topics` request will return the set of topics that have been recorded as active since a connector started or since the set of topics was reset for this connector. An additional endpoints allows users to reset the set of active topics for a connector via the second endpoint that this feature is adding: * A `PUT /connectors/{name}/topics/reset` request clears the set of active topics. An operator may enable or disable this feature by setting `topic.tracking.enable` (true by default). The `topic.tracking.enable` worker config property (true by default) allows an operator to enable/disable the entire feature. Or if the feature is enabled, the `topic.tracking.allow.reset` worker config property (true by default) allows an operator to control whether reset requests submitted to the Connect REST API are allowed. Author: Konstantine Karantasis <[email protected]> Reviewer: Randall Hauch <[email protected]>
- Loading branch information
1 parent
8494fdb
commit 7746301
Showing
20 changed files
with
778 additions
and
40 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
110 changes: 110 additions & 0 deletions
110
connect/runtime/src/main/java/org/apache/kafka/connect/runtime/TopicStatus.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,110 @@ | ||
/* | ||
* Licensed to the Apache Software Foundation (ASF) under one or more | ||
* contributor license agreements. See the NOTICE file distributed with | ||
* this work for additional information regarding copyright ownership. | ||
* The ASF licenses this file to You under the Apache License, Version 2.0 | ||
* (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 | ||
* | ||
* Unless required by applicable law or agreed to in writing, software | ||
* distributed under the License is distributed on an "AS IS" BASIS, | ||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. | ||
* See the License for the specific language governing permissions and | ||
* limitations under the License. | ||
*/ | ||
package org.apache.kafka.connect.runtime; | ||
|
||
import org.apache.kafka.connect.util.ConnectorTaskId; | ||
|
||
import java.util.Objects; | ||
|
||
/** | ||
* Represents the metadata that is stored as the value of the record that is stored in the | ||
* {@link org.apache.kafka.connect.storage.StatusBackingStore#put(TopicStatus)}, | ||
*/ | ||
public class TopicStatus { | ||
private final String topic; | ||
private final String connector; | ||
private final int task; | ||
private final long discoverTimestamp; | ||
|
||
public TopicStatus(String topic, ConnectorTaskId task, long discoverTimestamp) { | ||
this(topic, task.connector(), task.task(), discoverTimestamp); | ||
} | ||
|
||
public TopicStatus(String topic, String connector, int task, long discoverTimestamp) { | ||
this.topic = Objects.requireNonNull(topic); | ||
this.connector = Objects.requireNonNull(connector); | ||
this.task = task; | ||
this.discoverTimestamp = discoverTimestamp; | ||
} | ||
|
||
/** | ||
* Get the name of the topic. | ||
* | ||
* @return the topic name; never null | ||
*/ | ||
public String topic() { | ||
return topic; | ||
} | ||
|
||
/** | ||
* Get the name of the connector. | ||
* | ||
* @return the connector name; never null | ||
*/ | ||
public String connector() { | ||
return connector; | ||
} | ||
|
||
/** | ||
* Get the ID of the task that stored the topic status. | ||
* | ||
* @return the task ID | ||
*/ | ||
public int task() { | ||
return task; | ||
} | ||
|
||
/** | ||
* Get a timestamp that represents when this topic was discovered as being actively used by | ||
* this connector. | ||
* | ||
* @return the discovery timestamp | ||
*/ | ||
public long discoverTimestamp() { | ||
return discoverTimestamp; | ||
} | ||
|
||
@Override | ||
public String toString() { | ||
return "TopicStatus{" + | ||
"topic='" + topic + '\'' + | ||
", connector='" + connector + '\'' + | ||
", task=" + task + | ||
", discoverTimestamp=" + discoverTimestamp + | ||
'}'; | ||
} | ||
|
||
@Override | ||
public boolean equals(Object o) { | ||
if (this == o) { | ||
return true; | ||
} | ||
if (!(o instanceof TopicStatus)) { | ||
return false; | ||
} | ||
TopicStatus that = (TopicStatus) o; | ||
return task == that.task && | ||
discoverTimestamp == that.discoverTimestamp && | ||
topic.equals(that.topic) && | ||
connector.equals(that.connector); | ||
} | ||
|
||
@Override | ||
public int hashCode() { | ||
return Objects.hash(topic, connector, task, discoverTimestamp); | ||
} | ||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Oops, something went wrong.