Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi Viktor, Thanks for your comments! 1) I'm realizing now that there's some implicit behavior in the KIP that I should spell out explicitly. I was thinking that the timeout of 10 seconds that I mentioned in the "Public Interfaces" section would have to elapse before any 5xx responses were issued. So, if someone pinged the health endpoint while the worker was starting up, the worker would take up to 10 seconds to try to complete startup before giving up and responding to the request with a 503 status. This would obviate the need for a Retry-After header because it would apply a natural rate limiting to these requests during worker startup. Does this seem reasonable? We could diverge in behavior and only apply the 10 second timeout to the 500 case (i.e., worker has completed startup but is not live for other reasons), at which point a Retry-After header for the 503 case (worker still starting up) would make sense, but I can't think of any benefits to this approach. Thoughts? 2) As of KAFKA-15563 [1], we have some great infrastructure in place to identify worker actions that might be good candidates for the contents of this "worker events" topic. But I don't think conflating the retrieval of these events with the health endpoint is a good idea--IMO it should be a separate endpoint and the health endpoint should stay lightweight and simple. I'm also not sure it's necessary to expose the contents of this kind of topic via the REST API at all; we could instruct users to consume directly from the topic if they'd like to know the history of the worker. Overall it seems like a decent idea and I'd be happy to review a KIP for it, but like you mention, it seems like a pretty drastic change in scope and I don't think it needs to be included in this proposal. [1] - https://issues.apache.org/jira/browse/KAFKA-15563 Cheers, Chris On Tue, Jun 11, 2024 at 11:42 AM Viktor Somogyi-Vass wrote: > Hi Chris, > > I also have 2 other comments: > > 1. One more thing I came across is that should we provide the Retry-After > header in the response in case of 503 response? Although I'm not sure how > many clients honor this, we could add it just in case some does and if you > also find it useful. (We could default it to retry.backoff.ms.) > > 2. Adding to Adrian's comments, storing timestamped worker statuses in an > internal topic and then reading them from there would add valuable info > about what the worker does. For instance GET /health?startTime=45345323346 > could fetch events from the given timestamp additionally to your proposed > behavior. Also, if the rest server isn't available, it would serve in > itself as a log about the workers' behavior. I understand if you think it's > such a scope change that it should be an improvement KIP, but would like to > gauge what you think about this. > > Regards, > Viktor > > On Tue, Jun 11, 2024 at 4:34 PM Chris Egerton > wrote: > > > Hi Adrian, > > > > Thanks for your comments/questions! The responses to them are related so > > I'll try to address both at once. > > > > The most recent update I made to the KIP should help provide insight into > > what's going wrong if a non-200 response is returned. I don't plan on > > adding any structured data such as error codes or something like a > "phase" > > field with values like READING_CONFIG_TOPIC quite yet, but there is room > > for us to add human-readable information on the causes of failure in the > > "message" field (see KAFKA-15563 [1] and its PR [2] for an example of > what > > kind of information we might provide to users). Part of the problem is > that > > while I've heard plenty of (justified!) complaints about the Kafka > Connect > > REST API becoming unavailable and the difficulties users face with > > debugging their workers when that happens, I still don't feel we have a > > strong-enough grasp on the common causes for this scenario to commit to a > > response format that could be more machine-readable, and it can be > > surprisingly difficult to get to a root cause in some cases. > > > > I'm anticipating that users will rely on the endpoint primarily for two > > things: > > 1) Ensuring that a worker has completed startup successfully during a > > rolling upgrade (if you don't get a 200 after long enough, abort the > > upgrade, check the error message, and start investigating) > > 2) Ensuring that a worker remains healthy after it has joined the cluster > > (if you don't get a 200 for a sustained period of time, check the error > > message, and then decide whether to restart the process or issue a page) > > > > It's primarily designed to be easy to incorporate with automated tooling > > that has support for REST-based pr
Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi Chris, I also have 2 other comments: 1. One more thing I came across is that should we provide the Retry-After header in the response in case of 503 response? Although I'm not sure how many clients honor this, we could add it just in case some does and if you also find it useful. (We could default it to retry.backoff.ms.) 2. Adding to Adrian's comments, storing timestamped worker statuses in an internal topic and then reading them from there would add valuable info about what the worker does. For instance GET /health?startTime=45345323346 could fetch events from the given timestamp additionally to your proposed behavior. Also, if the rest server isn't available, it would serve in itself as a log about the workers' behavior. I understand if you think it's such a scope change that it should be an improvement KIP, but would like to gauge what you think about this. Regards, Viktor On Tue, Jun 11, 2024 at 4:34 PM Chris Egerton wrote: > Hi Adrian, > > Thanks for your comments/questions! The responses to them are related so > I'll try to address both at once. > > The most recent update I made to the KIP should help provide insight into > what's going wrong if a non-200 response is returned. I don't plan on > adding any structured data such as error codes or something like a "phase" > field with values like READING_CONFIG_TOPIC quite yet, but there is room > for us to add human-readable information on the causes of failure in the > "message" field (see KAFKA-15563 [1] and its PR [2] for an example of what > kind of information we might provide to users). Part of the problem is that > while I've heard plenty of (justified!) complaints about the Kafka Connect > REST API becoming unavailable and the difficulties users face with > debugging their workers when that happens, I still don't feel we have a > strong-enough grasp on the common causes for this scenario to commit to a > response format that could be more machine-readable, and it can be > surprisingly difficult to get to a root cause in some cases. > > I'm anticipating that users will rely on the endpoint primarily for two > things: > 1) Ensuring that a worker has completed startup successfully during a > rolling upgrade (if you don't get a 200 after long enough, abort the > upgrade, check the error message, and start investigating) > 2) Ensuring that a worker remains healthy after it has joined the cluster > (if you don't get a 200 for a sustained period of time, check the error > message, and then decide whether to restart the process or issue a page) > > It's primarily designed to be easy to incorporate with automated tooling > that has support for REST-based process health monitoring, while also > providing some human-readable information (when possible) if the worker > isn't healthy. This human-readable information should hopefully help people > gauge how to respond to non-200 responses, and we can try to improve > wording and granularity over time based on user feedback. You and other > users may develop automated responses based on the content of the error > messages, but beware that the wording may change across releases. > > Does that seem reasonable for V1 of this feature? I can definitely see room > for expansion of the response format in the future, but would like to hold > off on that for now. > > [1] - https://issues.apache.org/jira/browse/KAFKA-15563 > [2] - https://github.com/apache/kafka/pull/14562 > > Cheers, > > Chris > > On Tue, Jun 11, 2024 at 3:37 AM Adrian Preston > wrote: > > > Hi Chris, > > > > Good KIP – I think it will be very helpful in alerting and automating the > > resolution of common Connect problems. > > > > I have a couple of questions / suggestions: > > > > 1. What are you planning on documenting as guidance for using this new > > endpoint? My guess would be that if Connect doesn’t return a status of > 200 > > after some period I would either page someone, or restart the process? > But > > I’m missing the nuance of distinguishing between readiness and liveness, > is > > this for maintaining overall availability when rolling out updates to > > several Connect processes? > > > > 2. Would you consider providing a way to discover details about exactly > > which condition (or conditions) is/are failing? Perhaps by providing a > > richer JSON response? Something like this would help us adopt the health > > check, as we could start by paging someone for all failures, then over > time > > (as we gained confidence) transition more of the conditions over to being > > handled by automation. > > > > Regards, > > - Adrian > > > > > > From: Chris Egerton > > Date: Monday, 10 June 20
RE: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi Chris, Thanks for taking the time to provide more details about the intended usage. I hadn’t appreciated how nuanced (and perhaps in some cases not fully explored) the causes of an unhealthy Connect could be. With that in mind, I can see why you want to nail down a straight-forward and robust implementation before considering further enhancements. Cheers, - Adrian. From: Chris Egerton Date: Tuesday, 11 June 2024 at 15:34 To: dev@kafka.apache.org Subject: [EXTERNAL] Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect Hi Adrian, Thanks for your comments/questions! The responses to them are related so I'll try to address both at once. The most recent update I made to the KIP should help provide insight into what's going wrong if a non-200 response is returned. I don't plan on adding any structured data such as error codes or something like a "phase" field with values like READING_CONFIG_TOPIC quite yet, but there is room for us to add human-readable information on the causes of failure in the "message" field (see KAFKA-15563 [1] and its PR [2] for an example of what kind of information we might provide to users). Part of the problem is that while I've heard plenty of (justified!) complaints about the Kafka Connect REST API becoming unavailable and the difficulties users face with debugging their workers when that happens, I still don't feel we have a strong-enough grasp on the common causes for this scenario to commit to a response format that could be more machine-readable, and it can be surprisingly difficult to get to a root cause in some cases. I'm anticipating that users will rely on the endpoint primarily for two things: 1) Ensuring that a worker has completed startup successfully during a rolling upgrade (if you don't get a 200 after long enough, abort the upgrade, check the error message, and start investigating) 2) Ensuring that a worker remains healthy after it has joined the cluster (if you don't get a 200 for a sustained period of time, check the error message, and then decide whether to restart the process or issue a page) It's primarily designed to be easy to incorporate with automated tooling that has support for REST-based process health monitoring, while also providing some human-readable information (when possible) if the worker isn't healthy. This human-readable information should hopefully help people gauge how to respond to non-200 responses, and we can try to improve wording and granularity over time based on user feedback. You and other users may develop automated responses based on the content of the error messages, but beware that the wording may change across releases. Does that seem reasonable for V1 of this feature? I can definitely see room for expansion of the response format in the future, but would like to hold off on that for now. [1] - https://issues.apache.org/jira/browse/KAFKA-15563 [2] - https://github.com/apache/kafka/pull/14562 Cheers, Chris On Tue, Jun 11, 2024 at 3:37 AM Adrian Preston wrote: > Hi Chris, > > Good KIP – I think it will be very helpful in alerting and automating the > resolution of common Connect problems. > > I have a couple of questions / suggestions: > > 1. What are you planning on documenting as guidance for using this new > endpoint? My guess would be that if Connect doesn’t return a status of 200 > after some period I would either page someone, or restart the process? But > I’m missing the nuance of distinguishing between readiness and liveness, is > this for maintaining overall availability when rolling out updates to > several Connect processes? > > 2. Would you consider providing a way to discover details about exactly > which condition (or conditions) is/are failing? Perhaps by providing a > richer JSON response? Something like this would help us adopt the health > check, as we could start by paging someone for all failures, then over time > (as we gained confidence) transition more of the conditions over to being > handled by automation. > > Regards, > - Adrian > > > From: Chris Egerton > Date: Monday, 10 June 2024 at 15:26 > To: dev@kafka.apache.org > Subject: [EXTERNAL] Re: [DISCUSS] KIP-1017: A health check endpoint for > Kafka Connect > Hi all, > > Thanks for the positive feedback! > > I've made one small addition to the KIP since there's been a change to our > REST timeout error messages that's worth incorporating here. Quoting the > added section directly: > > > Note that the HTTP status codes and "status" fields in the JSON response > will match the exact examples above. However, the "message" field may be > augmented to include, among other things, more information about the > operation(s) the worker could be blocked on (such as was added in REST > timeout error messages in KAFKA-15563 [1]). > > Assuming this still
Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi Adrian, Thanks for your comments/questions! The responses to them are related so I'll try to address both at once. The most recent update I made to the KIP should help provide insight into what's going wrong if a non-200 response is returned. I don't plan on adding any structured data such as error codes or something like a "phase" field with values like READING_CONFIG_TOPIC quite yet, but there is room for us to add human-readable information on the causes of failure in the "message" field (see KAFKA-15563 [1] and its PR [2] for an example of what kind of information we might provide to users). Part of the problem is that while I've heard plenty of (justified!) complaints about the Kafka Connect REST API becoming unavailable and the difficulties users face with debugging their workers when that happens, I still don't feel we have a strong-enough grasp on the common causes for this scenario to commit to a response format that could be more machine-readable, and it can be surprisingly difficult to get to a root cause in some cases. I'm anticipating that users will rely on the endpoint primarily for two things: 1) Ensuring that a worker has completed startup successfully during a rolling upgrade (if you don't get a 200 after long enough, abort the upgrade, check the error message, and start investigating) 2) Ensuring that a worker remains healthy after it has joined the cluster (if you don't get a 200 for a sustained period of time, check the error message, and then decide whether to restart the process or issue a page) It's primarily designed to be easy to incorporate with automated tooling that has support for REST-based process health monitoring, while also providing some human-readable information (when possible) if the worker isn't healthy. This human-readable information should hopefully help people gauge how to respond to non-200 responses, and we can try to improve wording and granularity over time based on user feedback. You and other users may develop automated responses based on the content of the error messages, but beware that the wording may change across releases. Does that seem reasonable for V1 of this feature? I can definitely see room for expansion of the response format in the future, but would like to hold off on that for now. [1] - https://issues.apache.org/jira/browse/KAFKA-15563 [2] - https://github.com/apache/kafka/pull/14562 Cheers, Chris On Tue, Jun 11, 2024 at 3:37 AM Adrian Preston wrote: > Hi Chris, > > Good KIP – I think it will be very helpful in alerting and automating the > resolution of common Connect problems. > > I have a couple of questions / suggestions: > > 1. What are you planning on documenting as guidance for using this new > endpoint? My guess would be that if Connect doesn’t return a status of 200 > after some period I would either page someone, or restart the process? But > I’m missing the nuance of distinguishing between readiness and liveness, is > this for maintaining overall availability when rolling out updates to > several Connect processes? > > 2. Would you consider providing a way to discover details about exactly > which condition (or conditions) is/are failing? Perhaps by providing a > richer JSON response? Something like this would help us adopt the health > check, as we could start by paging someone for all failures, then over time > (as we gained confidence) transition more of the conditions over to being > handled by automation. > > Regards, > - Adrian > > > From: Chris Egerton > Date: Monday, 10 June 2024 at 15:26 > To: dev@kafka.apache.org > Subject: [EXTERNAL] Re: [DISCUSS] KIP-1017: A health check endpoint for > Kafka Connect > Hi all, > > Thanks for the positive feedback! > > I've made one small addition to the KIP since there's been a change to our > REST timeout error messages that's worth incorporating here. Quoting the > added section directly: > > > Note that the HTTP status codes and "status" fields in the JSON response > will match the exact examples above. However, the "message" field may be > augmented to include, among other things, more information about the > operation(s) the worker could be blocked on (such as was added in REST > timeout error messages in KAFKA-15563 [1]). > > Assuming this still looks okay to everyone, I'll kick off a vote thread > sometime this week or the next. > > [1] - https://issues.apache.org/jira/browse/KAFKA-15563 > > Cheers, > > Chris > > On Fri, Jun 7, 2024 at 12:01 PM Andrew Schofield < > andrew_schofi...@live.com> > wrote: > > > Hi Chris, > > This KIP looks good to me. I particularly like the explanation of how the > > result will specifically > > check the worker health in ways that have previously caused trouble. > > > > Thanks
RE: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi Chris, Good KIP – I think it will be very helpful in alerting and automating the resolution of common Connect problems. I have a couple of questions / suggestions: 1. What are you planning on documenting as guidance for using this new endpoint? My guess would be that if Connect doesn’t return a status of 200 after some period I would either page someone, or restart the process? But I’m missing the nuance of distinguishing between readiness and liveness, is this for maintaining overall availability when rolling out updates to several Connect processes? 2. Would you consider providing a way to discover details about exactly which condition (or conditions) is/are failing? Perhaps by providing a richer JSON response? Something like this would help us adopt the health check, as we could start by paging someone for all failures, then over time (as we gained confidence) transition more of the conditions over to being handled by automation. Regards, - Adrian From: Chris Egerton Date: Monday, 10 June 2024 at 15:26 To: dev@kafka.apache.org Subject: [EXTERNAL] Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect Hi all, Thanks for the positive feedback! I've made one small addition to the KIP since there's been a change to our REST timeout error messages that's worth incorporating here. Quoting the added section directly: > Note that the HTTP status codes and "status" fields in the JSON response will match the exact examples above. However, the "message" field may be augmented to include, among other things, more information about the operation(s) the worker could be blocked on (such as was added in REST timeout error messages in KAFKA-15563 [1]). Assuming this still looks okay to everyone, I'll kick off a vote thread sometime this week or the next. [1] - https://issues.apache.org/jira/browse/KAFKA-15563 Cheers, Chris On Fri, Jun 7, 2024 at 12:01 PM Andrew Schofield wrote: > Hi Chris, > This KIP looks good to me. I particularly like the explanation of how the > result will specifically > check the worker health in ways that have previously caused trouble. > > Thanks, > Andrew > > > On 7 Jun 2024, at 16:18, Mickael Maison > wrote: > > > > Hi Chris, > > > > Happy Friday! The KIP looks good to me. +1 > > > > Thanks, > > Mickael > > > > On Fri, Jan 26, 2024 at 8:41 PM Chris Egerton > wrote: > >> > >> Hi all, > >> > >> Happy Friday! I'd like to kick off discussion for KIP-1017, which (as > the > >> title suggests) proposes adding a health check endpoint for Kafka > Connect: > >> > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1017%3A+Health+check+endpoint+for+Kafka+Connect > >> > >> This is one of the longest-standing issues with Kafka Connect and I'm > >> hoping we can finally put it in the ground soon. Looking forward to > hearing > >> people's thoughts! > >> > >> Cheers, > >> > >> Chris > > Unless otherwise stated above: IBM United Kingdom Limited Registered in England and Wales with number 741598 Registered office: PO Box 41, North Harbour, Portsmouth, Hants. PO6 3AU
Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi all, Thanks for the positive feedback! I've made one small addition to the KIP since there's been a change to our REST timeout error messages that's worth incorporating here. Quoting the added section directly: > Note that the HTTP status codes and "status" fields in the JSON response will match the exact examples above. However, the "message" field may be augmented to include, among other things, more information about the operation(s) the worker could be blocked on (such as was added in REST timeout error messages in KAFKA-15563 [1]). Assuming this still looks okay to everyone, I'll kick off a vote thread sometime this week or the next. [1] - https://issues.apache.org/jira/browse/KAFKA-15563 Cheers, Chris On Fri, Jun 7, 2024 at 12:01 PM Andrew Schofield wrote: > Hi Chris, > This KIP looks good to me. I particularly like the explanation of how the > result will specifically > check the worker health in ways that have previously caused trouble. > > Thanks, > Andrew > > > On 7 Jun 2024, at 16:18, Mickael Maison > wrote: > > > > Hi Chris, > > > > Happy Friday! The KIP looks good to me. +1 > > > > Thanks, > > Mickael > > > > On Fri, Jan 26, 2024 at 8:41 PM Chris Egerton > wrote: > >> > >> Hi all, > >> > >> Happy Friday! I'd like to kick off discussion for KIP-1017, which (as > the > >> title suggests) proposes adding a health check endpoint for Kafka > Connect: > >> > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1017%3A+Health+check+endpoint+for+Kafka+Connect > >> > >> This is one of the longest-standing issues with Kafka Connect and I'm > >> hoping we can finally put it in the ground soon. Looking forward to > hearing > >> people's thoughts! > >> > >> Cheers, > >> > >> Chris > >
Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi Chris, This KIP looks good to me. I particularly like the explanation of how the result will specifically check the worker health in ways that have previously caused trouble. Thanks, Andrew > On 7 Jun 2024, at 16:18, Mickael Maison wrote: > > Hi Chris, > > Happy Friday! The KIP looks good to me. +1 > > Thanks, > Mickael > > On Fri, Jan 26, 2024 at 8:41 PM Chris Egerton wrote: >> >> Hi all, >> >> Happy Friday! I'd like to kick off discussion for KIP-1017, which (as the >> title suggests) proposes adding a health check endpoint for Kafka Connect: >> https://cwiki.apache.org/confluence/display/KAFKA/KIP-1017%3A+Health+check+endpoint+for+Kafka+Connect >> >> This is one of the longest-standing issues with Kafka Connect and I'm >> hoping we can finally put it in the ground soon. Looking forward to hearing >> people's thoughts! >> >> Cheers, >> >> Chris
Re: [DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi Chris, Happy Friday! The KIP looks good to me. +1 Thanks, Mickael On Fri, Jan 26, 2024 at 8:41 PM Chris Egerton wrote: > > Hi all, > > Happy Friday! I'd like to kick off discussion for KIP-1017, which (as the > title suggests) proposes adding a health check endpoint for Kafka Connect: > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1017%3A+Health+check+endpoint+for+Kafka+Connect > > This is one of the longest-standing issues with Kafka Connect and I'm > hoping we can finally put it in the ground soon. Looking forward to hearing > people's thoughts! > > Cheers, > > Chris
[jira] [Resolved] (KAFKA-16838) Kafka Connect loads old tasks from removed connectors
[ https://issues.apache.org/jira/browse/KAFKA-16838?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chris Egerton resolved KAFKA-16838. --- Fix Version/s: 3.9.0 Resolution: Fixed > Kafka Connect loads old tasks from removed connectors > - > > Key: KAFKA-16838 > URL: https://issues.apache.org/jira/browse/KAFKA-16838 > Project: Kafka > Issue Type: Bug > Components: connect >Affects Versions: 3.5.1, 3.6.1, 3.8.0 >Reporter: Sergey Ivanov >Assignee: Chris Egerton >Priority: Major > Fix For: 3.9.0 > > > Hello, > When creating connector we faced an error from one of our ConfigProviders > about not existing resource, but we didn't try to set that resource as config > value: > {code:java} > [2024-05-24T12:08:24.362][ERROR][request_id= ][tenant_id= > ][thread=DistributedHerder-connect-1-1][class=org.apache.kafka.connect.runtime.distributed.DistributedHerder][method=lambda$reconfigureConnectorTasksWithExponentialBackoffRetries$44] > [Worker clientId=connect-1, groupId=streaming-service_streaming_service] > Failed to reconfigure connector's tasks (local-file-sink), retrying after > backoff. > org.apache.kafka.common.config.ConfigException: Could not read properties > from file /opt/kafka/provider.properties > at > org.apache.kafka.common.config.provider.FileConfigProvider.get(FileConfigProvider.java:98) > at > org.apache.kafka.common.config.ConfigTransformer.transform(ConfigTransformer.java:103) > at > org.apache.kafka.connect.runtime.WorkerConfigTransformer.transform(WorkerConfigTransformer.java:58) > at > org.apache.kafka.connect.storage.ClusterConfigState.taskConfig(ClusterConfigState.java:181) > at > org.apache.kafka.connect.runtime.AbstractHerder.taskConfigsChanged(AbstractHerder.java:804) > at > org.apache.kafka.connect.runtime.distributed.DistributedHerder.publishConnectorTaskConfigs(DistributedHerder.java:2089) > at > org.apache.kafka.connect.runtime.distributed.DistributedHerder.reconfigureConnector(DistributedHerder.java:2082) > at > org.apache.kafka.connect.runtime.distributed.DistributedHerder.reconfigureConnectorTasksWithExponentialBackoffRetries(DistributedHerder.java:2025) > at > org.apache.kafka.connect.runtime.distributed.DistributedHerder.lambda$null$42(DistributedHerder.java:2038) > at > org.apache.kafka.connect.runtime.distributed.DistributedHerder.runRequest(DistributedHerder.java:2232) > at > org.apache.kafka.connect.runtime.distributed.DistributedHerder.tick(DistributedHerder.java:470) > at > org.apache.kafka.connect.runtime.distributed.DistributedHerder.run(DistributedHerder.java:371) > at > java.base/java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:539) > at java.base/java.util.concurrent.FutureTask.run(FutureTask.java:264) > at > java.base/java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1136) > at > java.base/java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:635) > at java.base/java.lang.Thread.run(Thread.java:840) > {code} > It looked like there already was connector with the same name and same > config, +but it wasn't.+ > After investigation we found out, that few months ago on that cloud there was > the connector with the same name and another value for config provider. Then > it was removed, but by some reason when we tried to create connector with the > same name months ago AbstractHerder tried to update tasks from our previous > connector > As an example I used FileConfigProvider, but actually any ConfigProvider is > acceptable which could raise exception if something wrong with config (like > result doesn't exist). > We continued our investigation and found the issue > https://issues.apache.org/jira/browse/KAFKA-7745 that says Connect doesn't > send tombstone message for *commit* and *task* records in the config topic of > Kafka Connect. As we remember, the config topic is `compact` *that means > commit and tasks are are always stored* (months, years after connector > removing) while tombstones for connector messages are cleaned with > {{delete.retention.ms}} property. That impacts further connector creations > with the same name. > We didn't investigate reasons in ConfigClusterStore and how to avoid that > issue, because would {+}like to ask{+}, probably it's better to fix > KAFKA-7745 and send tombstones for commit and task messages as connect does > for connector and target messages? > In the common way the TC looks like: > # Create connector with config provider to
[jira] [Resolved] (KAFKA-16837) Kafka Connect fails on update connector for incorrect previous Config Provider tasks
[ https://issues.apache.org/jira/browse/KAFKA-16837?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chris Egerton resolved KAFKA-16837. --- Fix Version/s: 3.9.0 Resolution: Fixed > Kafka Connect fails on update connector for incorrect previous Config > Provider tasks > > > Key: KAFKA-16837 > URL: https://issues.apache.org/jira/browse/KAFKA-16837 > Project: Kafka > Issue Type: Bug > Components: connect >Affects Versions: 3.5.1, 3.6.1, 3.8.0 >Reporter: Sergey Ivanov >Assignee: Chris Egerton >Priority: Major > Fix For: 3.9.0 > > Attachments: kafka_connect_config.png > > > Hello, > We faced an issue when is not possible to update Connector config if the > *previous* task contains ConfigProvider's value with incorrect value that > leads to ConfigException. > I can provide simple Test Case to reproduce it with FileConfigProvider, but > actually any ConfigProvider is acceptable that could raise exception if > something wrong with config (like resource doesn't exist). > *Prerequisites:* > Kafka Connect instance with config providers: > > {code:java} > config.providers=file > config.providers.file.class=org.apache.kafka.common.config.provider.FileConfigProvider{code} > > 1. Create Kafka topic "test" > 2. On the Kafka Connect instance create the file > "/opt/kafka/provider.properties" with content > {code:java} > topics=test > {code} > 3. Create simple FileSink connector: > {code:java} > PUT /connectors/local-file-sink/config > { > "connector.class": "FileStreamSink", > "tasks.max": "1", > "file": "/opt/kafka/test.sink.txt", > "topics": "${file:/opt/kafka/provider.properties:topics}" > } > {code} > 4. Checks that everything works fine: > {code:java} > GET /connectors?expand=info=status > ... > "status": { > "name": "local-file-sink", > "connector": { > "state": "RUNNING", > "worker_id": "10.10.10.10:8083" > }, > "tasks": [ > { > "id": 0, > "state": "RUNNING", > "worker_id": "10.10.10.10:8083" > } > ], > "type": "sink" > } > } > } > {code} > Looks fine. > 5. Renames the file to "/opt/kafka/provider2.properties". > 6. Update connector with new correct file name: > {code:java} > PUT /connectors/local-file-sink/config > { > "connector.class": "FileStreamSink", > "tasks.max": "1", > "file": "/opt/kafka/test.sink.txt", > "topics": "${file:/opt/kafka/provider2.properties:topics}" > } > {code} > Update {*}succeed{*}, got 200. > 7. Checks that everything works fine: > {code:java} > { > "local-file-sink": { > "info": { > "name": "local-file-sink", > "config": { > "connector.class": "FileStreamSink", > "file": "/opt/kafka/test.sink.txt", > "tasks.max": "1", > "topics": "${file:/opt/kafka/provider2.properties:topics}", > "name": "local-file-sink" > }, > "tasks": [ > { > "connector": "local-file-sink", > "task": 0 > } > ], > "type": "sink" > }, > "status": { > "name": "local-file-sink", > "connector": { > "state": "RUNNING", > "worker_id": "10.10.10.10:8083" > }, > "tasks": [ > { > "id": 0, > "state": "FAILED", > "worker_id": "10.10.10.10:8083", > "trace": "org.apache.kafka.common.errors.InvalidTopicException: > Invalid topics: [${file:/opt/kafka/provider.properties:topics}]" > } > ], > "type": "sink" > } > } > } > {code} > Config has been updated, but new task has not been c
[jira] [Resolved] (KAFKA-5451) Kafka Connect should scan classpath asynchronously
[ https://issues.apache.org/jira/browse/KAFKA-5451?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Greg Harris resolved KAFKA-5451. Resolution: Won't Do Kafka Connect needs the results of plugin scanning to answer basic REST queries, and to be assigned workloads via joining the group. Without finalized scan results, neither of these operations can meaningfully complete. Rather than make the scanning asynchronous, we have elected to make it faster via KAFKA-14627/KIP-898. It no longer makes sense to async-process something that takes <1s. > Kafka Connect should scan classpath asynchronously > -- > > Key: KAFKA-5451 > URL: https://issues.apache.org/jira/browse/KAFKA-5451 > Project: Kafka > Issue Type: Improvement > Components: connect >Affects Versions: 0.11.0.0 >Reporter: Randall Hauch >Assignee: Randall Hauch >Priority: Major > Labels: performance > Original Estimate: 24h > Remaining Estimate: 24h > > When Kafka Connect workers start up, they scan the classpath and module paths > for connectors, transformations, and converters. This takes anywhere from > 15-30sec or longer depending upon how many JARs are included. Currently, this > scanning is done synchronously during startup of the Kafka Connect workers, > even though the workers may not need the result of the scan. > The scanning logic should be asynchronous and should only block any > components that require the result of the scan. This will improve startup > time of the workers. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Resolved] (KAFKA-6208) Reduce startup time for Kafka Connect workers
[ https://issues.apache.org/jira/browse/KAFKA-6208?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Greg Harris resolved KAFKA-6208. Fix Version/s: 3.6.0 Resolution: Fixed This is fixed by setting plugin.discovery=service_load on 3.6+, see KAFKA-14627/KIP-898 for more details. > Reduce startup time for Kafka Connect workers > - > > Key: KAFKA-6208 > URL: https://issues.apache.org/jira/browse/KAFKA-6208 > Project: Kafka > Issue Type: Improvement > Components: connect >Affects Versions: 1.0.0 >Reporter: Randall Hauch >Priority: Major > Labels: needs-kip > Fix For: 3.6.0 > > > Kafka Connect startup times are excessive with a handful of connectors on the > plugin path or classpath. We should not be scanning three times (once for > connectors, once for SMTs, and once for converters), and hopefully we can > avoid scanning directories that are clearly not plugin directories. > We should also consider using Java's Service Loader to quickly identify > connectors. The latter would require a KIP and would require time to for > connectors to migrate, but we could be smarter about only scanning plugin > directories that need to be scanned. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (KAFKA-16838) Kafka Connect loads old tasks from removed connectors
Sergey Ivanov created KAFKA-16838: - Summary: Kafka Connect loads old tasks from removed connectors Key: KAFKA-16838 URL: https://issues.apache.org/jira/browse/KAFKA-16838 Project: Kafka Issue Type: Bug Components: connect Affects Versions: 3.6.1, 3.5.1, 3.8.0 Reporter: Sergey Ivanov Hello, When creating connector we faced an error from one of our ConfigProviders about not existing resource, but we didn't try to set that resource as config value: {code:java} [2024-05-24T12:08:24.362][ERROR][request_id= ][tenant_id= ][thread=DistributedHerder-connect-1-1][class=org.apache.kafka.connect.runtime.distributed.DistributedHerder][method=lambda$reconfigureConnectorTasksWithExponentialBackoffRetries$44] [Worker clientId=connect-1, groupId=streaming-service_streaming_service] Failed to reconfigure connector's tasks (local-file-sink), retrying after backoff. org.apache.kafka.common.config.ConfigException: Could not read properties from file /opt/kafka/provider.properties at org.apache.kafka.common.config.provider.FileConfigProvider.get(FileConfigProvider.java:98) at org.apache.kafka.common.config.ConfigTransformer.transform(ConfigTransformer.java:103) at org.apache.kafka.connect.runtime.WorkerConfigTransformer.transform(WorkerConfigTransformer.java:58) at org.apache.kafka.connect.storage.ClusterConfigState.taskConfig(ClusterConfigState.java:181) at org.apache.kafka.connect.runtime.AbstractHerder.taskConfigsChanged(AbstractHerder.java:804) at org.apache.kafka.connect.runtime.distributed.DistributedHerder.publishConnectorTaskConfigs(DistributedHerder.java:2089) at org.apache.kafka.connect.runtime.distributed.DistributedHerder.reconfigureConnector(DistributedHerder.java:2082) at org.apache.kafka.connect.runtime.distributed.DistributedHerder.reconfigureConnectorTasksWithExponentialBackoffRetries(DistributedHerder.java:2025) at org.apache.kafka.connect.runtime.distributed.DistributedHerder.lambda$null$42(DistributedHerder.java:2038) at org.apache.kafka.connect.runtime.distributed.DistributedHerder.runRequest(DistributedHerder.java:2232) at org.apache.kafka.connect.runtime.distributed.DistributedHerder.tick(DistributedHerder.java:470) at org.apache.kafka.connect.runtime.distributed.DistributedHerder.run(DistributedHerder.java:371) at java.base/java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:539) at java.base/java.util.concurrent.FutureTask.run(FutureTask.java:264) at java.base/java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1136) at java.base/java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:635) at java.base/java.lang.Thread.run(Thread.java:840) {code} After investigation we found out, that few months ago on that cloud there was the connector with the same name and another value for config provider. Then it was removed, but by some reason when we tried to create connector with the same name months ago AbstractHerder tried to update tasks from our previous connector As an example I use FileConfigProvider, but actually any ConfigProvider is accceptable which could raise exception if something wrong with config (like result doesn't exist). We continued our investigation and found the issue https://issues.apache.org/jira/browse/KAFKA-7745 that says Connect doesn't tombstone commit and task messages in the config topic of Kafka. As we remember config topic is `compact` *that means commit and tasks are stored every time (months, years after connector removing)* and impact further connector creations with the same name. We didn't investigate reasons in ConfigClusterStore and how to avoid that issue, because would {+}like to ask{+}, probably it's better to fix KAFKA-7745 and send tombstones for commit and task messages as connect does for connector and target messages? I have synthetic TC to reproduce that error if needed. This is linked with https://issues.apache.org/jira/browse/KAFKA-16837 but it's not the same issue. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[jira] [Created] (KAFKA-16837) Kafka Connect fails on update connector for incorrect previous Config Provider tasks
Sergey Ivanov created KAFKA-16837: - Summary: Kafka Connect fails on update connector for incorrect previous Config Provider tasks Key: KAFKA-16837 URL: https://issues.apache.org/jira/browse/KAFKA-16837 Project: Kafka Issue Type: Bug Components: connect Affects Versions: 3.6.1, 3.5.1, 3.8.0 Reporter: Sergey Ivanov Hello, We faced an issue when is not possible to update Connector config if the *previous* task contains ConfigProvider's value with incorrect value that leads to ConfigException. I can provide simple Test Case to reproduce it with FileConfigProvider, but actually any ConfigProvider is acceptable that could raise exception if something wrong with config (like resource doesn't exist). *Prerequisites:* Kafka Connect instance with config providers: {code:java} config.providers=file config.providers.file.class=org.apache.kafka.common.config.provider.FileConfigProvider{code} 1. Create Kafka topic "test" 2. On the KK instance create the file "/opt/kafka/provider.properties" with content {code:java} topics=test {code} 3. Create simple FileSink connector: {code:java} PUT /connectors/local-file-sink/config { "connector.class": "FileStreamSink", "tasks.max": "1", "file": "/opt/kafka/test.sink.txt", "topics": "${file:/opt/kafka/provider.properties:topics}" } {code} 4. Checks that everything works fine: {code:java} GET /connectors?expand=info=status ... "status": { "name": "local-file-sink", "connector": { "state": "RUNNING", "worker_id": "10.10.10.10:8083" }, "tasks": [ { "id": 0, "state": "RUNNING", "worker_id": "10.10.10.10:8083" } ], "type": "sink" } } } {code} Looks fine. 5. Renames the file to "/opt/kafka/provider2.properties". 6. Update connector with new correct file name: {code:java} PUT /connectors/local-file-sink/config { "connector.class": "FileStreamSink", "tasks.max": "1", "file": "/opt/kafka/test.sink.txt", "topics": "${file:/opt/kafka/provider2.properties:topics}" } {code} Update {*}succeed{*}, got 200. 7. Checks that everything works fine: {code:java} { "local-file-sink": { "info": { "name": "local-file-sink", "config": { "connector.class": "FileStreamSink", "file": "/opt/kafka/test.sink.txt", "tasks.max": "1", "topics": "${file:/opt/kafka/provider2.properties:topics}", "name": "local-file-sink" }, "tasks": [ { "connector": "local-file-sink", "task": 0 } ], "type": "sink" }, "status": { "name": "local-file-sink", "connector": { "state": "RUNNING", "worker_id": "10.10.10.10:8083" }, "tasks": [ { "id": 0, "state": "FAILED", "worker_id": "10.10.10.10:8083", "trace": "org.apache.kafka.common.errors.InvalidTopicException: Invalid topics: [${file:/opt/kafka/provider.properties:topics}]" } ], "type": "sink" } } } {code} Config has been updated, but new task has not been created. And as result connector doesn't work. It failed on: {code:java} [2024-05-24T12:08:24.362][ERROR][request_id= ][tenant_id= ][thread=DistributedHerder-connect-1-1][class=org.apache.kafka.connect.runtime.distributed.DistributedHerder][method=lambda$reconfigureConnectorTasksWithExponentialBackoffRetries$44] [Worker clientId=connect-1, groupId=streaming-service_streaming_service] Failed to reconfigure connector's tasks (local-file-sink), retrying after backoff. org.apache.kafka.common.config.ConfigException: Could not read properties from file /opt/kafka/provider.properties at org.apache.kafka.common.config.provider.FileConfigProvider.get(FileConfigProvider.java:98) at org.apache.kafka.common.config.ConfigTransformer.transform(ConfigTransformer.java:103) at org.apache.kafka.connect.runtime.WorkerConfigTransformer.transform(WorkerConfigTransformer.java:58) at org.apache.kafka.connect.storage.ClusterConfigState.taskConfig
[jira] [Resolved] (KAFKA-16603) Data loss when kafka connect sending data to Kafka
[ https://issues.apache.org/jira/browse/KAFKA-16603?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Chris Egerton resolved KAFKA-16603. --- Resolution: Not A Bug > Data loss when kafka connect sending data to Kafka > -- > > Key: KAFKA-16603 > URL: https://issues.apache.org/jira/browse/KAFKA-16603 > Project: Kafka > Issue Type: Bug > Components: clients, producer >Affects Versions: 3.3.1 >Reporter: Anil Dasari >Priority: Major > > We are experiencing a data loss when Kafka Source connector is failed to send > data to Kafka topic and offset topic. > Kafka cluster and Kafka connect details: > # Kafka connect version i.e client : Confluent community version 7.3.1 i.e > Kafka 3.3.1 > # Kafka version: 0.11.0 (server) > # Cluster size : 3 brokers > # Number of partitions in all topics = 3 > # Replication factor = 3 > # Min ISR set 2 > # Uses no transformations in Kafka connector > # Use default error tolerance i.e None. > Our connector checkpoints the offsets info received in > SourceTask#commitRecord and resume the data process from the persisted > checkpoint. > The data loss is noticed when broker is unresponsive for few mins due to high > load and kafka connector was restarted. Also, Kafka connector graceful > shutdown failed. > Logs: > > {code:java} > [Worker clientId=connect-1, > groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Discovered group > coordinator 10.75.100.176:31000 (id: 2147483647 rack: null) > Apr 22, 2024 @ 15:56:16.152 [Worker clientId=connect-1, > groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Group coordinator > 10.75.100.176:31000 (id: 2147483647 rack: null) is unavailable or invalid due > to cause: coordinator unavailable. isDisconnected: false. Rediscovery will be > attempted. > Apr 22, 2024 @ 15:56:16.153 [Worker clientId=connect-1, > groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Requesting disconnect from > last known coordinator 10.75.100.176:31000 (id: 2147483647 rack: null) > Apr 22, 2024 @ 15:56:16.514 [Worker clientId=connect-1, > groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Node 0 disconnected. > Apr 22, 2024 @ 15:56:16.708 [Producer > clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Node 0 > disconnected. > Apr 22, 2024 @ 15:56:16.710 [Worker clientId=connect-1, > groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Node 2147483647 > disconnected. > Apr 22, 2024 @ 15:56:16.731 [Worker clientId=connect-1, > groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Group coordinator > 10.75.100.176:31000 (id: 2147483647 rack: null) is unavailable or invalid due > to cause: coordinator unavailable. isDisconnected: true. Rediscovery will be > attempted. > Apr 22, 2024 @ 15:56:19.103 == Trying to sleep while stop == (** custom log > **) > Apr 22, 2024 @ 15:56:19.755 [Worker clientId=connect-1, > groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Broker coordinator was > unreachable for 3000ms. Revoking previous assignment Assignment{error=0, > leader='connect-1-8f41a1d2-6cc9-4956-9be3-1fbae9c6d305', > leaderUrl='http://10.75.100.46:8083/', offset=4, > connectorIds=[d094a5d7bbb046b99d62398cb84d648c], > taskIds=[d094a5d7bbb046b99d62398cb84d648c-0], revokedConnectorIds=[], > revokedTaskIds=[], delay=0} to avoid running tasks while not being a member > the group > Apr 22, 2024 @ 15:56:19.866 Stopping connector > d094a5d7bbb046b99d62398cb84d648c > Apr 22, 2024 @ 15:56:19.874 Stopping task d094a5d7bbb046b99d62398cb84d648c-0 > Apr 22, 2024 @ 15:56:19.880 Scheduled shutdown for > WorkerConnectorWorkerConnector{id=d094a5d7bbb046b99d62398cb84d648c} > Apr 22, 2024 @ 15:56:24.105 Connector 'd094a5d7bbb046b99d62398cb84d648c' > failed to properly shut down, has become unresponsive, and may be consuming > external resources. Correct the configuration for this connector or remove > the connector. After fixing the connector, it may be necessary to restart > this worker to release any consumed resources. > Apr 22, 2024 @ 15:56:24.110 [Producer > clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Closing the > Kafka producer with timeoutMillis = 0 ms. > Apr 22, 2024 @ 15:56:24.110 [Producer > clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Proceeding to > force close the producer since pending requests could not be completed within > timeout 0 ms. > Apr 22, 2024 @ 15:56:24.112 [Producer > clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Beginning > shutdown of Kafka producer I/O thread, sending remaining records.
[jira] [Created] (KAFKA-16603) Data loss when kafka connect sending data to Kafka
Anil Dasari created KAFKA-16603: --- Summary: Data loss when kafka connect sending data to Kafka Key: KAFKA-16603 URL: https://issues.apache.org/jira/browse/KAFKA-16603 Project: Kafka Issue Type: Bug Components: clients, producer Affects Versions: 3.3.1 Reporter: Anil Dasari We are experiencing a data loss when Kafka Source connector is failed to send data to Kafka topic and offset topic. Kafka cluster and Kafka connect details: # Kafka version : Confluent community version 7.3.1 i.e Kafka 3.3.1 # Cluster size : 3 brokers # Number of partitions in all topics = 3 # Replication factor = 3 # Min ISR set 2 # Uses no transformations in Kafka connector # Use default error tolerance i.e None. Our connector checkpoints the offsets info received in SourceTask#commitRecord and resume the data process from the persisted checkpoint. The data loss is noticed when broker is unresponsive for few mins due to high load and kafka connector was restarted. However, Kafka connector graceful shutdown failed. Logs: {code:java} [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Discovered group coordinator 10.75.100.176:31000 (id: 2147483647 rack: null) Apr 22, 2024 @ 15:56:16.152 [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Group coordinator 10.75.100.176:31000 (id: 2147483647 rack: null) is unavailable or invalid due to cause: coordinator unavailable. isDisconnected: false. Rediscovery will be attempted. Apr 22, 2024 @ 15:56:16.153 [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Requesting disconnect from last known coordinator 10.75.100.176:31000 (id: 2147483647 rack: null) Apr 22, 2024 @ 15:56:16.514 [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Node 0 disconnected. Apr 22, 2024 @ 15:56:16.708 [Producer clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Node 0 disconnected. Apr 22, 2024 @ 15:56:16.710 [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Node 2147483647 disconnected. Apr 22, 2024 @ 15:56:16.731 [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Group coordinator 10.75.100.176:31000 (id: 2147483647 rack: null) is unavailable or invalid due to cause: coordinator unavailable. isDisconnected: true. Rediscovery will be attempted. Apr 22, 2024 @ 15:56:19.103 == Trying to sleep while stop == (** custom log **) Apr 22, 2024 @ 15:56:19.755 [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Broker coordinator was unreachable for 3000ms. Revoking previous assignment Assignment{error=0, leader='connect-1-8f41a1d2-6cc9-4956-9be3-1fbae9c6d305', leaderUrl='http://10.75.100.46:8083/', offset=4, connectorIds=[d094a5d7bbb046b99d62398cb84d648c], taskIds=[d094a5d7bbb046b99d62398cb84d648c-0], revokedConnectorIds=[], revokedTaskIds=[], delay=0} to avoid running tasks while not being a member the group Apr 22, 2024 @ 15:56:19.866 Stopping connector d094a5d7bbb046b99d62398cb84d648c Apr 22, 2024 @ 15:56:19.874 Stopping task d094a5d7bbb046b99d62398cb84d648c-0 Apr 22, 2024 @ 15:56:19.880 Scheduled shutdown for WorkerConnectorWorkerConnector{id=d094a5d7bbb046b99d62398cb84d648c} Apr 22, 2024 @ 15:56:24.105 Connector 'd094a5d7bbb046b99d62398cb84d648c' failed to properly shut down, has become unresponsive, and may be consuming external resources. Correct the configuration for this connector or remove the connector. After fixing the connector, it may be necessary to restart this worker to release any consumed resources. Apr 22, 2024 @ 15:56:24.110 [Producer clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Closing the Kafka producer with timeoutMillis = 0 ms. Apr 22, 2024 @ 15:56:24.110 [Producer clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Proceeding to force close the producer since pending requests could not be completed within timeout 0 ms. Apr 22, 2024 @ 15:56:24.112 [Producer clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Beginning shutdown of Kafka producer I/O thread, sending remaining records. Apr 22, 2024 @ 15:56:24.112 [Producer clientId=connector-producer-d094a5d7bbb046b99d62398cb84d648c-0] Aborting incomplete batches due to forced shutdown Apr 22, 2024 @ 15:56:24.113 WorkerSourceTaskWorkerSourceTask{id=d094a5d7bbb046b99d62398cb84d648c-0} Committing offsets Apr 22, 2024 @ 15:56:24.113 WorkerSourceTaskWorkerSourceTask{id=d094a5d7bbb046b99d62398cb84d648c-0} Either no records were produced by the task since the last offset commit, or every record has been filtered out by a transformation or dropped due to transformation or conversion errors. Apr 22, 2024 @ 15:56:24.146 [Worker clientId=connect-1, groupId=pg-group-adf06ea08abb4394ad4f2787481fee17] Finished stopping tasks
[jira] [Created] (KAFKA-16326) Kafka Connect unable to find javax dependency on Quarkus update to 3.X
Pau Ortega Puig created KAFKA-16326: --- Summary: Kafka Connect unable to find javax dependency on Quarkus update to 3.X Key: KAFKA-16326 URL: https://issues.apache.org/jira/browse/KAFKA-16326 Project: Kafka Issue Type: Bug Components: connect Affects Versions: 3.6.0 Reporter: Pau Ortega Puig We have a repository that uses both Quarkus and Kafka Connect. We're trying to update Quarkus to version 3.X but we're finding an error when configuring Kafka Connect: {code:java} java.lang.ClassNotFoundException: javax.ws.rs.core.Configurable{code} We are aware of the _javax_ to _jakarta_ libraries change and indeed we have changed all our direct dependencies to use {_}jakarta{_}. It looks like Kafka Connect still uses _javax_ dependencies and at runtime it is unable to find them. We attach a minimal repo that reproduces the issue here: [https://github.com/pauortegathoughtworks/quarkus-kafka-connect-bug] Also we provide the full stack trace here: {code:java} java.lang.RuntimeException: Failed to start quarkus at io.quarkus.runner.ApplicationImpl.doStart(Unknown Source) at io.quarkus.runtime.Application.start(Application.java:101) at io.quarkus.runtime.ApplicationLifecycleManager.run(ApplicationLifecycleManager.java:111) at io.quarkus.runtime.Quarkus.run(Quarkus.java:71) at io.quarkus.runtime.Quarkus.run(Quarkus.java:44) at io.quarkus.runtime.Quarkus.run(Quarkus.java:124) at io.quarkus.runner.GeneratedMain.main(Unknown Source) at java.base/jdk.internal.reflect.NativeMethodAccessorImpl.invoke0(Native Method) at java.base/jdk.internal.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:77) at java.base/jdk.internal.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) at java.base/java.lang.reflect.Method.invoke(Method.java:568) at io.quarkus.runner.bootstrap.StartupActionImpl$1.run(StartupActionImpl.java:113) at java.base/java.lang.Thread.run(Thread.java:833) Caused by: java.lang.NoClassDefFoundError: javax/ws/rs/core/Configurable at org.apache.kafka.connect.cli.AbstractConnectCli.startConnect(AbstractConnectCli.java:128) at org.acme.KafkaConnectRunner.start(KafkaConnectRunner.java:88) at org.acme.KafkaConnectRunner.onStart(KafkaConnectRunner.java:73) at org.acme.KafkaConnectRunner_Observer_onStart_1_-42pHN04Og1MUKiGWhJM7NweE.notify(Unknown Source) at io.quarkus.arc.impl.EventImpl$Notifier.notifyObservers(EventImpl.java:346) at io.quarkus.arc.impl.EventImpl$Notifier.notify(EventImpl.java:328) at io.quarkus.arc.impl.EventImpl.fire(EventImpl.java:82) at io.quarkus.arc.runtime.ArcRecorder.fireLifecycleEvent(ArcRecorder.java:157) at io.quarkus.arc.runtime.ArcRecorder.handleLifecycleEvents(ArcRecorder.java:108) at io.quarkus.deployment.steps.LifecycleEventsBuildStep$startupEvent1144526294.deploy_0(Unknown Source) at io.quarkus.deployment.steps.LifecycleEventsBuildStep$startupEvent1144526294.deploy(Unknown Source) ... 13 more Caused by: java.lang.ClassNotFoundException: javax.ws.rs.core.Configurable at java.base/jdk.internal.loader.BuiltinClassLoader.loadClass(BuiltinClassLoader.java:641) at java.base/jdk.internal.loader.ClassLoaders$AppClassLoader.loadClass(ClassLoaders.java:188) at java.base/java.lang.ClassLoader.loadClass(ClassLoader.java:520) at io.quarkus.bootstrap.classloading.QuarkusClassLoader.loadClass(QuarkusClassLoader.java:518) at io.quarkus.bootstrap.classloading.QuarkusClassLoader.loadClass(QuarkusClassLoader.java:468) ... 24 more {code} -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi team, Excuse me for duplicate. The last message isn't attached to the thread. I try to fix it by this email. Ping for review / vote for KIP-967 [1]. Voting thread is here [2] Please pay your attention, I beg of you. I'm a little frustrated by the lack of community interest in what seems like a simple and necessary patch. Is the SslEngineFactory refactoring requirement blocking? [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi team, Ping for review / vote for KIP-967 [1]. Voting thread is here [2] Please pay your attention, I beg of you. I'm a little frustrated by the lack of community interest in what seems like a simple and necessary patch. Is the SslEngineFactory refactoring requirement blocking? [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw -- With best regards, Taras Ledkov
[DISCUSS] KIP-1017: A health check endpoint for Kafka Connect
Hi all, Happy Friday! I'd like to kick off discussion for KIP-1017, which (as the title suggests) proposes adding a health check endpoint for Kafka Connect: https://cwiki.apache.org/confluence/display/KAFKA/KIP-1017%3A+Health+check+endpoint+for+Kafka+Connect This is one of the longest-standing issues with Kafka Connect and I'm hoping we can finally put it in the ground soon. Looking forward to hearing people's thoughts! Cheers, Chris
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi all, The vote for KIP-1004 passes with the following +1 votes and no +0 or -1 votes: - Hector Geraldino - Mickael Maison (binding) - Greg Harris (binding) - Yash Mayya (binding) - Federico Valeri With regards to the open discussion about whether to remove the deprecated tasks.max.enforce property in 4.0.0 or later, I've tweaked the KIP to clearly state that it may take place in 4.0.0 but may also be delayed. A deprecated property does not require a KIP for removal, so we have some wiggle room should the discussion continue, especially if people feel strongly that we should push to remove it in time for 4.0.0. Thanks all for your votes and discussion! Cheers, Chris On Fri, Jan 5, 2024 at 3:45 PM Greg Harris wrote: > Hey Chris, > > Thanks for keeping KIP-987 in-mind. > > The current design of KIP-987 doesn't take tasks.max.enforce into > account, but I think it may be possible to only allow the protocol > upgrade when tasks.max.enforce is true if we were to try to enforce > it. It may also be reasonable to just have a warning about it appended > to the documentation string for tasks.max.enforce. > I am fine with either keeping or removing it in 4.0, leaning towards > keeping it, for the same reasons you listed above. > > Thanks! > Greg > > On Fri, Jan 5, 2024 at 9:40 AM Chris Egerton > wrote: > > > > Hi Yash, > > > > Thanks for raising the possibility of a more aggressive removal schedule > > for the tasks.max.enforce property now that it seems a 3.8.x branch is > > likely--I was wondering if someone would bring that up! > > > > I think I'd prefer to err on the side of caution and give users more time > > to adjust, since some may skip 3.8.x and upgrade to 4.0.x, 4.1.x, etc. > > directly instead. It seems like the maintenance cost will be fairly low, > > and with the option to programmatically require it to be set to true in > > order to work with other features we may want to develop in the future, > it > > shouldn't block any progress in the meantime. Thoughts? I'd also be > curious > > what Greg Harris thinks about this, given that it seems relevant to > KIP-987 > > [1]. > > > > [1] - > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-987%3A+Connect+Static+Assignments > > > > Cheers, > > > > Chris > > > > On Thu, Jan 4, 2024 at 4:45 AM Federico Valeri > wrote: > > > > > Thanks! This will finally reconcile Javadoc and implementation. > > > > > > +1 (non binding) > > > > > > On Thu, Jan 4, 2024 at 6:49 AM Yash Mayya > wrote: > > > > > > > > Hi Chris, > > > > > > > > +1 (binding), thanks for the KIP. > > > > > > > > Based on discussion in other threads, it looks like the community is > > > > aligned with having a 3.8 release before the 4.0 release so we > should be > > > > able to remove the 'tasks.max.enforce' connector property in 4.0 > (we'd > > > > discussed potentially having to live with this property until 5.0 in > this > > > > KIP's discussion thread). Once we have confirmation of a 3.8 release, > > > will > > > > this KIP be updated to reflect the exact AK versions where the > deprecated > > > > property will be introduced and removed? > > > > > > > > Thanks, > > > > Yash > > > > > > > > On Wed, Jan 3, 2024 at 11:37 PM Greg Harris > > > > > > > > wrote: > > > > > > > > > Hey Chris, > > > > > > > > > > Thanks for the KIP! I think the aggressive default and deprecation > > > > > schedule is the right choice for this change. > > > > > > > > > > +1 (binding) > > > > > > > > > > On Wed, Jan 3, 2024 at 9:01 AM Mickael Maison < > > > mickael.mai...@gmail.com> > > > > > wrote: > > > > > > > > > > > > Hi Chris, > > > > > > > > > > > > +1 (binding), thanks for the KIP > > > > > > > > > > > > Mickael > > > > > > > > > > > > On Tue, Jan 2, 2024 at 8:55 PM Hector Geraldino (BLOOMBERG/ 919 > 3RD > > > A) > > > > > > wrote: > > > > > > > > > > > > > > +1 (non-binding) > > > > > > > > > > > > > > Thanks Chris! > > > > > > > > > > > > > > From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: > > > > > de
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hey Chris, Thanks for keeping KIP-987 in-mind. The current design of KIP-987 doesn't take tasks.max.enforce into account, but I think it may be possible to only allow the protocol upgrade when tasks.max.enforce is true if we were to try to enforce it. It may also be reasonable to just have a warning about it appended to the documentation string for tasks.max.enforce. I am fine with either keeping or removing it in 4.0, leaning towards keeping it, for the same reasons you listed above. Thanks! Greg On Fri, Jan 5, 2024 at 9:40 AM Chris Egerton wrote: > > Hi Yash, > > Thanks for raising the possibility of a more aggressive removal schedule > for the tasks.max.enforce property now that it seems a 3.8.x branch is > likely--I was wondering if someone would bring that up! > > I think I'd prefer to err on the side of caution and give users more time > to adjust, since some may skip 3.8.x and upgrade to 4.0.x, 4.1.x, etc. > directly instead. It seems like the maintenance cost will be fairly low, > and with the option to programmatically require it to be set to true in > order to work with other features we may want to develop in the future, it > shouldn't block any progress in the meantime. Thoughts? I'd also be curious > what Greg Harris thinks about this, given that it seems relevant to KIP-987 > [1]. > > [1] - > https://cwiki.apache.org/confluence/display/KAFKA/KIP-987%3A+Connect+Static+Assignments > > Cheers, > > Chris > > On Thu, Jan 4, 2024 at 4:45 AM Federico Valeri wrote: > > > Thanks! This will finally reconcile Javadoc and implementation. > > > > +1 (non binding) > > > > On Thu, Jan 4, 2024 at 6:49 AM Yash Mayya wrote: > > > > > > Hi Chris, > > > > > > +1 (binding), thanks for the KIP. > > > > > > Based on discussion in other threads, it looks like the community is > > > aligned with having a 3.8 release before the 4.0 release so we should be > > > able to remove the 'tasks.max.enforce' connector property in 4.0 (we'd > > > discussed potentially having to live with this property until 5.0 in this > > > KIP's discussion thread). Once we have confirmation of a 3.8 release, > > will > > > this KIP be updated to reflect the exact AK versions where the deprecated > > > property will be introduced and removed? > > > > > > Thanks, > > > Yash > > > > > > On Wed, Jan 3, 2024 at 11:37 PM Greg Harris > > > > > wrote: > > > > > > > Hey Chris, > > > > > > > > Thanks for the KIP! I think the aggressive default and deprecation > > > > schedule is the right choice for this change. > > > > > > > > +1 (binding) > > > > > > > > On Wed, Jan 3, 2024 at 9:01 AM Mickael Maison < > > mickael.mai...@gmail.com> > > > > wrote: > > > > > > > > > > Hi Chris, > > > > > > > > > > +1 (binding), thanks for the KIP > > > > > > > > > > Mickael > > > > > > > > > > On Tue, Jan 2, 2024 at 8:55 PM Hector Geraldino (BLOOMBERG/ 919 3RD > > A) > > > > > wrote: > > > > > > > > > > > > +1 (non-binding) > > > > > > > > > > > > Thanks Chris! > > > > > > > > > > > > From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: > > > > dev@kafka.apache.org > > > > > > Subject: Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka > > > > Connect > > > > > > > > > > > > Hi all, > > > > > > > > > > > > Happy New Year! Wanted to give this a bump now that the holidays > > are > > > > over > > > > > > for a lot of us. Looking forward to people's thoughts! > > > > > > > > > > > > Cheers, > > > > > > > > > > > > Chris > > > > > > > > > > > > On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton > > wrote: > > > > > > > > > > > > > Hi all, > > > > > > > > > > > > > > I'd like to call for a vote on KIP-1004, which adds enforcement > > for > > > > the > > > > > > > tasks.max connector property in Kafka Connect. > > > > > > > > > > > > > > The KIP: > > > > > > > > > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > > > > > property+in+Kafka+Connect > > > > > > > > > > > > > > The discussion thread: > > > > > > > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > > > > > > > > > > > > > Cheers, > > > > > > > > > > > > > > Chris > > > > > > > > > > > > > > > > > > > > > > > > >
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi Yash, Thanks for raising the possibility of a more aggressive removal schedule for the tasks.max.enforce property now that it seems a 3.8.x branch is likely--I was wondering if someone would bring that up! I think I'd prefer to err on the side of caution and give users more time to adjust, since some may skip 3.8.x and upgrade to 4.0.x, 4.1.x, etc. directly instead. It seems like the maintenance cost will be fairly low, and with the option to programmatically require it to be set to true in order to work with other features we may want to develop in the future, it shouldn't block any progress in the meantime. Thoughts? I'd also be curious what Greg Harris thinks about this, given that it seems relevant to KIP-987 [1]. [1] - https://cwiki.apache.org/confluence/display/KAFKA/KIP-987%3A+Connect+Static+Assignments Cheers, Chris On Thu, Jan 4, 2024 at 4:45 AM Federico Valeri wrote: > Thanks! This will finally reconcile Javadoc and implementation. > > +1 (non binding) > > On Thu, Jan 4, 2024 at 6:49 AM Yash Mayya wrote: > > > > Hi Chris, > > > > +1 (binding), thanks for the KIP. > > > > Based on discussion in other threads, it looks like the community is > > aligned with having a 3.8 release before the 4.0 release so we should be > > able to remove the 'tasks.max.enforce' connector property in 4.0 (we'd > > discussed potentially having to live with this property until 5.0 in this > > KIP's discussion thread). Once we have confirmation of a 3.8 release, > will > > this KIP be updated to reflect the exact AK versions where the deprecated > > property will be introduced and removed? > > > > Thanks, > > Yash > > > > On Wed, Jan 3, 2024 at 11:37 PM Greg Harris > > > wrote: > > > > > Hey Chris, > > > > > > Thanks for the KIP! I think the aggressive default and deprecation > > > schedule is the right choice for this change. > > > > > > +1 (binding) > > > > > > On Wed, Jan 3, 2024 at 9:01 AM Mickael Maison < > mickael.mai...@gmail.com> > > > wrote: > > > > > > > > Hi Chris, > > > > > > > > +1 (binding), thanks for the KIP > > > > > > > > Mickael > > > > > > > > On Tue, Jan 2, 2024 at 8:55 PM Hector Geraldino (BLOOMBERG/ 919 3RD > A) > > > > wrote: > > > > > > > > > > +1 (non-binding) > > > > > > > > > > Thanks Chris! > > > > > > > > > > From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: > > > dev@kafka.apache.org > > > > > Subject: Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka > > > Connect > > > > > > > > > > Hi all, > > > > > > > > > > Happy New Year! Wanted to give this a bump now that the holidays > are > > > over > > > > > for a lot of us. Looking forward to people's thoughts! > > > > > > > > > > Cheers, > > > > > > > > > > Chris > > > > > > > > > > On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton > wrote: > > > > > > > > > > > Hi all, > > > > > > > > > > > > I'd like to call for a vote on KIP-1004, which adds enforcement > for > > > the > > > > > > tasks.max connector property in Kafka Connect. > > > > > > > > > > > > The KIP: > > > > > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > > > > property+in+Kafka+Connect > > > > > > > > > > > > The discussion thread: > > > > > > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > > > > > > > > > > > Cheers, > > > > > > > > > > > > Chris > > > > > > > > > > > > > > > > > > > >
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Thanks! This will finally reconcile Javadoc and implementation. +1 (non binding) On Thu, Jan 4, 2024 at 6:49 AM Yash Mayya wrote: > > Hi Chris, > > +1 (binding), thanks for the KIP. > > Based on discussion in other threads, it looks like the community is > aligned with having a 3.8 release before the 4.0 release so we should be > able to remove the 'tasks.max.enforce' connector property in 4.0 (we'd > discussed potentially having to live with this property until 5.0 in this > KIP's discussion thread). Once we have confirmation of a 3.8 release, will > this KIP be updated to reflect the exact AK versions where the deprecated > property will be introduced and removed? > > Thanks, > Yash > > On Wed, Jan 3, 2024 at 11:37 PM Greg Harris > wrote: > > > Hey Chris, > > > > Thanks for the KIP! I think the aggressive default and deprecation > > schedule is the right choice for this change. > > > > +1 (binding) > > > > On Wed, Jan 3, 2024 at 9:01 AM Mickael Maison > > wrote: > > > > > > Hi Chris, > > > > > > +1 (binding), thanks for the KIP > > > > > > Mickael > > > > > > On Tue, Jan 2, 2024 at 8:55 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) > > > wrote: > > > > > > > > +1 (non-binding) > > > > > > > > Thanks Chris! > > > > > > > > From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: > > dev@kafka.apache.org > > > > Subject: Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka > > Connect > > > > > > > > Hi all, > > > > > > > > Happy New Year! Wanted to give this a bump now that the holidays are > > over > > > > for a lot of us. Looking forward to people's thoughts! > > > > > > > > Cheers, > > > > > > > > Chris > > > > > > > > On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton wrote: > > > > > > > > > Hi all, > > > > > > > > > > I'd like to call for a vote on KIP-1004, which adds enforcement for > > the > > > > > tasks.max connector property in Kafka Connect. > > > > > > > > > > The KIP: > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > > > property+in+Kafka+Connect > > > > > > > > > > The discussion thread: > > > > > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > > > > > > > > > Cheers, > > > > > > > > > > Chris > > > > > > > > > > > > > > >
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi Chris, +1 (binding), thanks for the KIP. Based on discussion in other threads, it looks like the community is aligned with having a 3.8 release before the 4.0 release so we should be able to remove the 'tasks.max.enforce' connector property in 4.0 (we'd discussed potentially having to live with this property until 5.0 in this KIP's discussion thread). Once we have confirmation of a 3.8 release, will this KIP be updated to reflect the exact AK versions where the deprecated property will be introduced and removed? Thanks, Yash On Wed, Jan 3, 2024 at 11:37 PM Greg Harris wrote: > Hey Chris, > > Thanks for the KIP! I think the aggressive default and deprecation > schedule is the right choice for this change. > > +1 (binding) > > On Wed, Jan 3, 2024 at 9:01 AM Mickael Maison > wrote: > > > > Hi Chris, > > > > +1 (binding), thanks for the KIP > > > > Mickael > > > > On Tue, Jan 2, 2024 at 8:55 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) > > wrote: > > > > > > +1 (non-binding) > > > > > > Thanks Chris! > > > > > > From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: > dev@kafka.apache.org > > > Subject: Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka > Connect > > > > > > Hi all, > > > > > > Happy New Year! Wanted to give this a bump now that the holidays are > over > > > for a lot of us. Looking forward to people's thoughts! > > > > > > Cheers, > > > > > > Chris > > > > > > On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton wrote: > > > > > > > Hi all, > > > > > > > > I'd like to call for a vote on KIP-1004, which adds enforcement for > the > > > > tasks.max connector property in Kafka Connect. > > > > > > > > The KIP: > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > > property+in+Kafka+Connect > > > > > > > > The discussion thread: > > > > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > > > > > > > Cheers, > > > > > > > > Chris > > > > > > > > > > >
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hey Chris, Thanks for the KIP! I think the aggressive default and deprecation schedule is the right choice for this change. +1 (binding) On Wed, Jan 3, 2024 at 9:01 AM Mickael Maison wrote: > > Hi Chris, > > +1 (binding), thanks for the KIP > > Mickael > > On Tue, Jan 2, 2024 at 8:55 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) > wrote: > > > > +1 (non-binding) > > > > Thanks Chris! > > > > From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: > > dev@kafka.apache.org > > Subject: Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect > > > > Hi all, > > > > Happy New Year! Wanted to give this a bump now that the holidays are over > > for a lot of us. Looking forward to people's thoughts! > > > > Cheers, > > > > Chris > > > > On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton wrote: > > > > > Hi all, > > > > > > I'd like to call for a vote on KIP-1004, which adds enforcement for the > > > tasks.max connector property in Kafka Connect. > > > > > > The KIP: > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > property+in+Kafka+Connect > > > > > > The discussion thread: > > > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > > > > > Cheers, > > > > > > Chris > > > > > > >
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi Chris, +1 (binding), thanks for the KIP Mickael On Tue, Jan 2, 2024 at 8:55 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) wrote: > > +1 (non-binding) > > Thanks Chris! > > From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: > dev@kafka.apache.org > Subject: Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect > > Hi all, > > Happy New Year! Wanted to give this a bump now that the holidays are over > for a lot of us. Looking forward to people's thoughts! > > Cheers, > > Chris > > On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton wrote: > > > Hi all, > > > > I'd like to call for a vote on KIP-1004, which adds enforcement for the > > tasks.max connector property in Kafka Connect. > > > > The KIP: > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > property+in+Kafka+Connect > > > > The discussion thread: > > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > > > Cheers, > > > > Chris > > > >
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
+1 (non-binding) Thanks Chris! From: dev@kafka.apache.org At: 01/02/24 11:49:18 UTC-5:00To: dev@kafka.apache.org Subject: Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect Hi all, Happy New Year! Wanted to give this a bump now that the holidays are over for a lot of us. Looking forward to people's thoughts! Cheers, Chris On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton wrote: > Hi all, > > I'd like to call for a vote on KIP-1004, which adds enforcement for the > tasks.max connector property in Kafka Connect. > > The KIP: > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ property+in+Kafka+Connect > > The discussion thread: > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > Cheers, > > Chris >
Re: [VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi all, Happy New Year! Wanted to give this a bump now that the holidays are over for a lot of us. Looking forward to people's thoughts! Cheers, Chris On Mon, Dec 4, 2023 at 10:36 AM Chris Egerton wrote: > Hi all, > > I'd like to call for a vote on KIP-1004, which adds enforcement for the > tasks.max connector property in Kafka Connect. > > The KIP: > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+property+in+Kafka+Connect > > The discussion thread: > https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d > > Cheers, > > Chris >
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi team, Ping for review / vote for KIP-967 [1]. Voting thread is here [2] [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw -- With best regards, Taras Ledkov On Wed, Dec 6, 2023 at 2:21 PM Taras Ledkov wrote: > > Hi Greg, > > > Taras, are you interested in dynamic SSL reconfiguration in Connect? > > Would you be willing to investigate the details of that for the KIP? > I would prefer to separate this functionality into the next KIP. But > if you / (the community) consider it necessary to combine this in one > KIP/patch, then I’m ready to do it. > > The entry point for dynamic reconfiguration for Connect is not clear for me. > For a broker it is the DynamicConfig functionality. Should we add a > REST endpoint to reconfigure Connect or use another way? > > -- > With best regards, > Taras Ledkov > > On Tue, Dec 5, 2023 at 2:42 AM Greg Harris > wrote: > > > > Hi Chris, > > > > Thank you for your comments above. I disagree with your recommendation > > for a new SslEngineFactory variant/hierarchy. > > > > 1. A superinterface could be more confusing to users. Since this is an > > interface in `clients`, the connect-specific interface would also need > > to be in clients, despite being superfluous for clients users and > > broker developers. > > 2. A user could implement the reduced interface, and then have issues > > loading their implementation in a broker, and need to find > > documentation/javadocs to explain to them why. > > 3. This interface has been in use since 2020, so I don't believe that > > the burden of implementing these methods has been excessive. I assume > > there's at least one "static" implementation out there that would have > > benefitted from the proposed super-interface, but they managed to > > adapt to the standardized interface. > > 4. Implementations that don't want to provide basic implementations > > can throw UnsupportedOperationException from the extra methods as a > > last resort. > > > > On the other hand, how much would it take to support the full suite of > > SslEngineFactory operations in Connect? Could part of this KIP be > > improving Connect to make use of these methods? This would help unify > > the experience for users that implement the interface specifically for > > the dynamic reconfiguration support, and rely on it on the broker > > side. > > > > Taras, are you interested in dynamic SSL reconfiguration in Connect? > > Would you be willing to investigate the details of that for the KIP? > > > > Thanks, > > Greg > > > > On Mon, Dec 4, 2023 at 1:17 PM Chris Egerton > > wrote: > > > > > > Hi Taras, > > > > > > Regarding slimming down the interface: IMO, we should do this right the > > > first time, and that includes not requiring unnecessary methods from > > > users. > > > I think BaseSslEngineFactory is good enough as a superinterface. > > > > > > > > > Regarding the parsing logic: I think the KIP needs to be more explicit. We > > > should add something like this to the proposed changes section: > > > > > > "If any properties are present in the worker config with a prefix of > > > "listeners.https.", then only properties with that prefix will be passed > > > to > > > the SSL engine factory. Otherwise, all top-level worker properties will be > > > passed to the SSL engine factory. Note that this differs slightly from > > > existing logic in that the set of properties (prefixed or otherwise) will > > > not be filtered based on a predefined set of keys; this will enable custom > > > SSL engine factories to define and accept custom properties." > > > > > > I also took a quick look at the prototype (I usually try not to do this > > > since we vote on KIP documents, not PRs). I don't think we should populate > > > default values for SSL-related properties before sending properties to the > > > SSL engine factory, since it may confuse users who have written custom SSL > > > engine factories to see that properties not specified in their Connect > > > worker config are being passed to their factory. Instead, the default SSL > > > engine factory used by Connect can perform this logic, and we can let > > > other > > > custom factories be responsible for their own default values. > > > > > > > > > Cheers, > > > > > > Chris > > > > > > On Wed, Nov 29, 2023 at 8:36 AM Taras Ledkov wrote: > > > > > > > Hi team, > > > > > > > > Ping for review / vote for KIP-967 [1]. > > > > Voting thread is here [2] > > > > > > > > [1]. > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer > > > > [2]. https://github.com/apache/kafka/pull/14203 > > > > [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw > > > > > > > > -- > > > > With best regards, > > > > Taras Ledkov > > > >
[DISCUSS] Kafka Connect source task interruption semantics
Hi all, I'd like to solicit input from users and maintainers on a problem we've been dealing with for source task cleanup logic. If you'd like to pore over some Jira history, here's the primary link: https://issues.apache.org/jira/browse/KAFKA-15090 To summarize, we accidentally introduced a breaking change for Kafka Connect in https://github.com/apache/kafka/pull/9669. Before that change, the SourceTask::stop method [1] would be invoked on a separate thread from the one that did the actual data processing for the task (polling the task for records, transforming and converting those records, then sending them to Kafka). After that change, we began invoking SourceTask::stop on the same thread that handled data processing for the task. This had the effect that tasks which blocked indefinitely in the SourceTask::poll method [2] with the expectation that they could stop blocking when SourceTask::stop was invoked would no longer be capable of graceful shutdown, and may even hang forever. This breaking change was introduced in the 3.0.0 release, a little over two three ago. Since then, source connectors may have been modified to adapt to the change in behavior by the Connect framework. As a result, we are hesitant to go back to the prior logic of invoking SourceTask::stop on a separate thread (see the linked Jira ticket for more detail on this front). In https://github.com/apache/kafka/pull/14316, I proposed that we begin interrupting the data processing thread for the source task after it had exhausted its graceful shutdown timeout (i.e., when the Kafka Connect runtime decides to cancel [3], [4], [5] the task). I believe this change is fairly non-controversial--once a task has failed to shut down gracefully, the runtime can and should do whatever it wants to force a shutdown, graceful or otherwise. With all that context out of the way, the question I'd like to ask is: do we believe it's also appropriate to interrupt the data processing thread when the task is scheduled for shutdown [6], [7]? This interruption would ideally be followed up by a graceful shutdown of the task, which may require the Kafka Connect runtime to handle a potential InterruptedException from SourceTask::poll. Other exceptions (such as a wrapped InterruptedException) would be impossible to handle gracefully, and may lead to spurious error messages in the logs and failed final offset commits for connectors that do not work well with this new behavior. Finally, one important note: in the official documentation for SourceTask::poll, we do already state that this method should not block for too long: > If no data is currently available, this method should block but return control to the caller regularly (by returning null) in order for the task to transition to the PAUSED state if requested to do so. Looking forward to everyone's thoughts on this tricky issue! Cheers, Chris [1] - https://kafka.apache.org/36/javadoc/org/apache/kafka/connect/source/SourceTask.html#stop() [2] - https://kafka.apache.org/36/javadoc/org/apache/kafka/connect/source/SourceTask.html#poll() [3] - https://github.com/apache/kafka/blob/c5ee82cab447b094ad7491200afa319515d5f467/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java#L1037 [4] - https://github.com/apache/kafka/blob/c5ee82cab447b094ad7491200afa319515d5f467/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/WorkerTask.java#L129-L136 [5] - https://github.com/apache/kafka/blob/c5ee82cab447b094ad7491200afa319515d5f467/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/AbstractWorkerSourceTask.java#L284-L297 [6] - https://github.com/apache/kafka/blob/c5ee82cab447b094ad7491200afa319515d5f467/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java#L1014 [7] - https://github.com/apache/kafka/blob/c5ee82cab447b094ad7491200afa319515d5f467/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/WorkerTask.java#L112-L127
[jira] [Created] (KAFKA-15988) Kafka Connect OffsetsApiIntegrationTest takes too long
Chris Egerton created KAFKA-15988: - Summary: Kafka Connect OffsetsApiIntegrationTest takes too long Key: KAFKA-15988 URL: https://issues.apache.org/jira/browse/KAFKA-15988 Project: Kafka Issue Type: Improvement Components: KafkaConnect Reporter: Chris Egerton Assignee: Chris Egerton The [OffsetsApiIntegrationTest suite|https://github.com/apache/kafka/blob/c515bf51f820f26ff6be6b0fde03b47b69a10b00/connect/runtime/src/test/java/org/apache/kafka/connect/integration/OffsetsApiIntegrationTest.java] currently contains 27 test cases. Each test case begins by creating embedded Kafka and Kafka Connect clusters, which is fairly resource-intensive and time-consuming. If possible, we should reuse those embedded clusters across test cases. -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Greg, > Taras, are you interested in dynamic SSL reconfiguration in Connect? > Would you be willing to investigate the details of that for the KIP? I would prefer to separate this functionality into the next KIP. But if you / (the community) consider it necessary to combine this in one KIP/patch, then I’m ready to do it. The entry point for dynamic reconfiguration for Connect is not clear for me. For a broker it is the DynamicConfig functionality. Should we add a REST endpoint to reconfigure Connect or use another way? -- With best regards, Taras Ledkov On Tue, Dec 5, 2023 at 2:42 AM Greg Harris wrote: > > Hi Chris, > > Thank you for your comments above. I disagree with your recommendation > for a new SslEngineFactory variant/hierarchy. > > 1. A superinterface could be more confusing to users. Since this is an > interface in `clients`, the connect-specific interface would also need > to be in clients, despite being superfluous for clients users and > broker developers. > 2. A user could implement the reduced interface, and then have issues > loading their implementation in a broker, and need to find > documentation/javadocs to explain to them why. > 3. This interface has been in use since 2020, so I don't believe that > the burden of implementing these methods has been excessive. I assume > there's at least one "static" implementation out there that would have > benefitted from the proposed super-interface, but they managed to > adapt to the standardized interface. > 4. Implementations that don't want to provide basic implementations > can throw UnsupportedOperationException from the extra methods as a > last resort. > > On the other hand, how much would it take to support the full suite of > SslEngineFactory operations in Connect? Could part of this KIP be > improving Connect to make use of these methods? This would help unify > the experience for users that implement the interface specifically for > the dynamic reconfiguration support, and rely on it on the broker > side. > > Taras, are you interested in dynamic SSL reconfiguration in Connect? > Would you be willing to investigate the details of that for the KIP? > > Thanks, > Greg > > On Mon, Dec 4, 2023 at 1:17 PM Chris Egerton wrote: > > > > Hi Taras, > > > > Regarding slimming down the interface: IMO, we should do this right the > > first time, and that includes not requiring unnecessary methods from users. > > I think BaseSslEngineFactory is good enough as a superinterface. > > > > > > Regarding the parsing logic: I think the KIP needs to be more explicit. We > > should add something like this to the proposed changes section: > > > > "If any properties are present in the worker config with a prefix of > > "listeners.https.", then only properties with that prefix will be passed to > > the SSL engine factory. Otherwise, all top-level worker properties will be > > passed to the SSL engine factory. Note that this differs slightly from > > existing logic in that the set of properties (prefixed or otherwise) will > > not be filtered based on a predefined set of keys; this will enable custom > > SSL engine factories to define and accept custom properties." > > > > I also took a quick look at the prototype (I usually try not to do this > > since we vote on KIP documents, not PRs). I don't think we should populate > > default values for SSL-related properties before sending properties to the > > SSL engine factory, since it may confuse users who have written custom SSL > > engine factories to see that properties not specified in their Connect > > worker config are being passed to their factory. Instead, the default SSL > > engine factory used by Connect can perform this logic, and we can let other > > custom factories be responsible for their own default values. > > > > > > Cheers, > > > > Chris > > > > On Wed, Nov 29, 2023 at 8:36 AM Taras Ledkov wrote: > > > > > Hi team, > > > > > > Ping for review / vote for KIP-967 [1]. > > > Voting thread is here [2] > > > > > > [1]. > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer > > > [2]. https://github.com/apache/kafka/pull/14203 > > > [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw > > > > > > -- > > > With best regards, > > > Taras Ledkov > > >
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Chris, > I don't think we should populate default values for SSL-related properties > before sending properties to the SSL engine factory, > since it may confuse users who have written custom SSL engine factories > to see that properties not specified in their Connect worker config are being > passed to their factory. I slightly disagree with this.. The behavior implemented in PR is consistent with SSL configuration for clients and brokers. This behavior does not prevent the implementation of custom SSL factories for the broker and client. But you are right about reviewing PR and KIP. Let's focus on the KIP. -- With best regards, Taras Ledkov On Tue, Dec 5, 2023 at 12:18 AM Chris Egerton wrote: > > Hi Taras, > > Regarding slimming down the interface: IMO, we should do this right the > first time, and that includes not requiring unnecessary methods from users. > I think BaseSslEngineFactory is good enough as a superinterface. > > > Regarding the parsing logic: I think the KIP needs to be more explicit. We > should add something like this to the proposed changes section: > > "If any properties are present in the worker config with a prefix of > "listeners.https.", then only properties with that prefix will be passed to > the SSL engine factory. Otherwise, all top-level worker properties will be > passed to the SSL engine factory. Note that this differs slightly from > existing logic in that the set of properties (prefixed or otherwise) will > not be filtered based on a predefined set of keys; this will enable custom > SSL engine factories to define and accept custom properties." > > I also took a quick look at the prototype (I usually try not to do this > since we vote on KIP documents, not PRs). I don't think we should populate > default values for SSL-related properties before sending properties to the > SSL engine factory, since it may confuse users who have written custom SSL > engine factories to see that properties not specified in their Connect > worker config are being passed to their factory. Instead, the default SSL > engine factory used by Connect can perform this logic, and we can let other > custom factories be responsible for their own default values. > > > Cheers, > > Chris > > On Wed, Nov 29, 2023 at 8:36 AM Taras Ledkov wrote: > > > Hi team, > > > > Ping for review / vote for KIP-967 [1]. > > Voting thread is here [2] > > > > [1]. > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer > > [2]. https://github.com/apache/kafka/pull/14203 > > [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw > > > > -- > > With best regards, > > Taras Ledkov > >
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Chris, Thank you for your comments above. I disagree with your recommendation for a new SslEngineFactory variant/hierarchy. 1. A superinterface could be more confusing to users. Since this is an interface in `clients`, the connect-specific interface would also need to be in clients, despite being superfluous for clients users and broker developers. 2. A user could implement the reduced interface, and then have issues loading their implementation in a broker, and need to find documentation/javadocs to explain to them why. 3. This interface has been in use since 2020, so I don't believe that the burden of implementing these methods has been excessive. I assume there's at least one "static" implementation out there that would have benefitted from the proposed super-interface, but they managed to adapt to the standardized interface. 4. Implementations that don't want to provide basic implementations can throw UnsupportedOperationException from the extra methods as a last resort. On the other hand, how much would it take to support the full suite of SslEngineFactory operations in Connect? Could part of this KIP be improving Connect to make use of these methods? This would help unify the experience for users that implement the interface specifically for the dynamic reconfiguration support, and rely on it on the broker side. Taras, are you interested in dynamic SSL reconfiguration in Connect? Would you be willing to investigate the details of that for the KIP? Thanks, Greg On Mon, Dec 4, 2023 at 1:17 PM Chris Egerton wrote: > > Hi Taras, > > Regarding slimming down the interface: IMO, we should do this right the > first time, and that includes not requiring unnecessary methods from users. > I think BaseSslEngineFactory is good enough as a superinterface. > > > Regarding the parsing logic: I think the KIP needs to be more explicit. We > should add something like this to the proposed changes section: > > "If any properties are present in the worker config with a prefix of > "listeners.https.", then only properties with that prefix will be passed to > the SSL engine factory. Otherwise, all top-level worker properties will be > passed to the SSL engine factory. Note that this differs slightly from > existing logic in that the set of properties (prefixed or otherwise) will > not be filtered based on a predefined set of keys; this will enable custom > SSL engine factories to define and accept custom properties." > > I also took a quick look at the prototype (I usually try not to do this > since we vote on KIP documents, not PRs). I don't think we should populate > default values for SSL-related properties before sending properties to the > SSL engine factory, since it may confuse users who have written custom SSL > engine factories to see that properties not specified in their Connect > worker config are being passed to their factory. Instead, the default SSL > engine factory used by Connect can perform this logic, and we can let other > custom factories be responsible for their own default values. > > > Cheers, > > Chris > > On Wed, Nov 29, 2023 at 8:36 AM Taras Ledkov wrote: > > > Hi team, > > > > Ping for review / vote for KIP-967 [1]. > > Voting thread is here [2] > > > > [1]. > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer > > [2]. https://github.com/apache/kafka/pull/14203 > > [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw > > > > -- > > With best regards, > > Taras Ledkov > >
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Taras, Regarding slimming down the interface: IMO, we should do this right the first time, and that includes not requiring unnecessary methods from users. I think BaseSslEngineFactory is good enough as a superinterface. Regarding the parsing logic: I think the KIP needs to be more explicit. We should add something like this to the proposed changes section: "If any properties are present in the worker config with a prefix of "listeners.https.", then only properties with that prefix will be passed to the SSL engine factory. Otherwise, all top-level worker properties will be passed to the SSL engine factory. Note that this differs slightly from existing logic in that the set of properties (prefixed or otherwise) will not be filtered based on a predefined set of keys; this will enable custom SSL engine factories to define and accept custom properties." I also took a quick look at the prototype (I usually try not to do this since we vote on KIP documents, not PRs). I don't think we should populate default values for SSL-related properties before sending properties to the SSL engine factory, since it may confuse users who have written custom SSL engine factories to see that properties not specified in their Connect worker config are being passed to their factory. Instead, the default SSL engine factory used by Connect can perform this logic, and we can let other custom factories be responsible for their own default values. Cheers, Chris On Wed, Nov 29, 2023 at 8:36 AM Taras Ledkov wrote: > Hi team, > > Ping for review / vote for KIP-967 [1]. > Voting thread is here [2] > > [1]. > https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer > [2]. https://github.com/apache/kafka/pull/14203 > [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw > > -- > With best regards, > Taras Ledkov >
Re: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi all, It seems like there are no objections to this KIP, so I've kicked off a vote thread: https://lists.apache.org/thread/dgq332o5j25rwddbvfydf7ttrclldw17 Cheers, Chris On Fri, Nov 24, 2023 at 10:39 PM Chris Egerton wrote: > Hi Yash, > > Thanks for taking a look! Yeah, it looks like we'll be forced to hold onto > the property until 5.0 if we can't introduce it at least one minor release > before 4.0. I don't think this is the worst thing. Although it'd be nice to > have the property completely removed when designing features like KIP-987, > if necessary, we can also declare any new features incompatible with > connectors that have opted out of enforcement of the tasks.max property > (and likely enforce this restraint programmatically via preflight > validation, failing connectors/tasks, log messages, etc.). > > Cheers, > > Chris > > On Wed, Nov 22, 2023 at 10:04 PM Yash Mayya wrote: > > > Hi Chris, > > > > Thanks for the well written and comprehensive KIP! Given that we're > already > > past the KIP freeze deadline for 3.7.0 ( > > https://cwiki.apache.org/confluence/display/KAFKA/Release+Plan+3.7.0) > and > > there may not be a 3.8.0 release before the 4.0.0 release, would we then > be > > forced to punt the removal of "tasks.max.enforce" to a future 5.0.0 > > release? I don't have any other comments, and the proposed changes make > > sense to me. > > > > Thanks, > > Yash > > > > On Mon, Nov 20, 2023 at 10:50 PM Chris Egerton > > wrote: > > > > > Hi Hector, > > > > > > Thanks for taking a look! I think the key difference between the > proposed > > > behavior and the rejected alternative is that the set of tasks that > will > > be > > > running with the former is still a complete set of tasks, whereas the > set > > > of tasks in the latter is a subset of tasks. Also noteworthy but > slightly > > > less important: the problem will be more visible to users with the > former > > > (the connector will still be marked FAILED) than with the latter. > > > > > > Cheers, > > > > > > Chris > > > > > > On Tue, Nov 21, 2023, 00:53 Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > > hgerald...@bloomberg.net> wrote: > > > > > > > Thanks for the KIP Chris, adding this check makes total sense. > > > > > > > > I do have one question. The second paragraph in the Public Interfaces > > > > section states: > > > > > > > > "If the connector generated excessive tasks after being reconfigured, > > > then > > > > any existing tasks for the connector will be allowed to continue > > running, > > > > unless that existing set of tasks also exceeds the tasks.max > property." > > > > > > > > Would not failing the connector land us in the second scenario of > > > > 'Rejected Alternatives'? > > > > > > > > From: dev@kafka.apache.org At: 11/11/23 00:27:44 UTC-5:00To: > > > > dev@kafka.apache.org > > > > Subject: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka > > Connect > > > > > > > > Hi all, > > > > > > > > I'd like to open up KIP-1004 for discussion: > > > > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > > > property+in+Kafka+Connect > > > > > > > > As a brief summary: this KIP proposes that the Kafka Connect runtime > > > start > > > > failing connectors that generate a greater number of tasks than the > > > > tasks.max property, with an optional emergency override that can be > > used > > > to > > > > continue running these (probably-buggy) connectors if absolutely > > > necessary. > > > > > > > > I'll be taking time off most of the next three weeks, so response > > latency > > > > may be a bit higher than usual, but I wanted to kick off the > discussion > > > in > > > > case we can land this in time for the upcoming 3.7.0 release. > > > > > > > > Cheers, > > > > > > > > Chris > > > > > > > > > > > > > > > > > >
[VOTE] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi all, I'd like to call for a vote on KIP-1004, which adds enforcement for the tasks.max connector property in Kafka Connect. The KIP: https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+property+in+Kafka+Connect The discussion thread: https://lists.apache.org/thread/scx75cjwm19jyt19wxky41q9smf5nx6d Cheers, Chris
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi team, Ping for review / vote for KIP-967 [1]. Voting thread is here [2] [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer [2]. https://github.com/apache/kafka/pull/14203 [2]. https://lists.apache.org/thread/wc4v5v3pynl15g1q547m2croqsqgmzpw -- With best regards, Taras Ledkov
Re: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi Yash, Thanks for taking a look! Yeah, it looks like we'll be forced to hold onto the property until 5.0 if we can't introduce it at least one minor release before 4.0. I don't think this is the worst thing. Although it'd be nice to have the property completely removed when designing features like KIP-987, if necessary, we can also declare any new features incompatible with connectors that have opted out of enforcement of the tasks.max property (and likely enforce this restraint programmatically via preflight validation, failing connectors/tasks, log messages, etc.). Cheers, Chris On Wed, Nov 22, 2023 at 10:04 PM Yash Mayya wrote: > Hi Chris, > > Thanks for the well written and comprehensive KIP! Given that we're already > past the KIP freeze deadline for 3.7.0 ( > https://cwiki.apache.org/confluence/display/KAFKA/Release+Plan+3.7.0) and > there may not be a 3.8.0 release before the 4.0.0 release, would we then be > forced to punt the removal of "tasks.max.enforce" to a future 5.0.0 > release? I don't have any other comments, and the proposed changes make > sense to me. > > Thanks, > Yash > > On Mon, Nov 20, 2023 at 10:50 PM Chris Egerton > wrote: > > > Hi Hector, > > > > Thanks for taking a look! I think the key difference between the proposed > > behavior and the rejected alternative is that the set of tasks that will > be > > running with the former is still a complete set of tasks, whereas the set > > of tasks in the latter is a subset of tasks. Also noteworthy but slightly > > less important: the problem will be more visible to users with the former > > (the connector will still be marked FAILED) than with the latter. > > > > Cheers, > > > > Chris > > > > On Tue, Nov 21, 2023, 00:53 Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > hgerald...@bloomberg.net> wrote: > > > > > Thanks for the KIP Chris, adding this check makes total sense. > > > > > > I do have one question. The second paragraph in the Public Interfaces > > > section states: > > > > > > "If the connector generated excessive tasks after being reconfigured, > > then > > > any existing tasks for the connector will be allowed to continue > running, > > > unless that existing set of tasks also exceeds the tasks.max property." > > > > > > Would not failing the connector land us in the second scenario of > > > 'Rejected Alternatives'? > > > > > > From: dev@kafka.apache.org At: 11/11/23 00:27:44 UTC-5:00To: > > > dev@kafka.apache.org > > > Subject: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka > Connect > > > > > > Hi all, > > > > > > I'd like to open up KIP-1004 for discussion: > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > > property+in+Kafka+Connect > > > > > > As a brief summary: this KIP proposes that the Kafka Connect runtime > > start > > > failing connectors that generate a greater number of tasks than the > > > tasks.max property, with an optional emergency override that can be > used > > to > > > continue running these (probably-buggy) connectors if absolutely > > necessary. > > > > > > I'll be taking time off most of the next three weeks, so response > latency > > > may be a bit higher than usual, but I wanted to kick off the discussion > > in > > > case we can land this in time for the upcoming 3.7.0 release. > > > > > > Cheers, > > > > > > Chris > > > > > > > > > > > >
Re: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi Chris, Thanks for the well written and comprehensive KIP! Given that we're already past the KIP freeze deadline for 3.7.0 ( https://cwiki.apache.org/confluence/display/KAFKA/Release+Plan+3.7.0) and there may not be a 3.8.0 release before the 4.0.0 release, would we then be forced to punt the removal of "tasks.max.enforce" to a future 5.0.0 release? I don't have any other comments, and the proposed changes make sense to me. Thanks, Yash On Mon, Nov 20, 2023 at 10:50 PM Chris Egerton wrote: > Hi Hector, > > Thanks for taking a look! I think the key difference between the proposed > behavior and the rejected alternative is that the set of tasks that will be > running with the former is still a complete set of tasks, whereas the set > of tasks in the latter is a subset of tasks. Also noteworthy but slightly > less important: the problem will be more visible to users with the former > (the connector will still be marked FAILED) than with the latter. > > Cheers, > > Chris > > On Tue, Nov 21, 2023, 00:53 Hector Geraldino (BLOOMBERG/ 919 3RD A) < > hgerald...@bloomberg.net> wrote: > > > Thanks for the KIP Chris, adding this check makes total sense. > > > > I do have one question. The second paragraph in the Public Interfaces > > section states: > > > > "If the connector generated excessive tasks after being reconfigured, > then > > any existing tasks for the connector will be allowed to continue running, > > unless that existing set of tasks also exceeds the tasks.max property." > > > > Would not failing the connector land us in the second scenario of > > 'Rejected Alternatives'? > > > > From: dev@kafka.apache.org At: 11/11/23 00:27:44 UTC-5:00To: > > dev@kafka.apache.org > > Subject: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect > > > > Hi all, > > > > I'd like to open up KIP-1004 for discussion: > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > > property+in+Kafka+Connect > > > > As a brief summary: this KIP proposes that the Kafka Connect runtime > start > > failing connectors that generate a greater number of tasks than the > > tasks.max property, with an optional emergency override that can be used > to > > continue running these (probably-buggy) connectors if absolutely > necessary. > > > > I'll be taking time off most of the next three weeks, so response latency > > may be a bit higher than usual, but I wanted to kick off the discussion > in > > case we can land this in time for the upcoming 3.7.0 release. > > > > Cheers, > > > > Chris > > > > > > >
Re: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi Hector, Thanks for taking a look! I think the key difference between the proposed behavior and the rejected alternative is that the set of tasks that will be running with the former is still a complete set of tasks, whereas the set of tasks in the latter is a subset of tasks. Also noteworthy but slightly less important: the problem will be more visible to users with the former (the connector will still be marked FAILED) than with the latter. Cheers, Chris On Tue, Nov 21, 2023, 00:53 Hector Geraldino (BLOOMBERG/ 919 3RD A) < hgerald...@bloomberg.net> wrote: > Thanks for the KIP Chris, adding this check makes total sense. > > I do have one question. The second paragraph in the Public Interfaces > section states: > > "If the connector generated excessive tasks after being reconfigured, then > any existing tasks for the connector will be allowed to continue running, > unless that existing set of tasks also exceeds the tasks.max property." > > Would not failing the connector land us in the second scenario of > 'Rejected Alternatives'? > > From: dev@kafka.apache.org At: 11/11/23 00:27:44 UTC-5:00To: > dev@kafka.apache.org > Subject: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect > > Hi all, > > I'd like to open up KIP-1004 for discussion: > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ > property+in+Kafka+Connect > > As a brief summary: this KIP proposes that the Kafka Connect runtime start > failing connectors that generate a greater number of tasks than the > tasks.max property, with an optional emergency override that can be used to > continue running these (probably-buggy) connectors if absolutely necessary. > > I'll be taking time off most of the next three weeks, so response latency > may be a bit higher than usual, but I wanted to kick off the discussion in > case we can land this in time for the upcoming 3.7.0 release. > > Cheers, > > Chris > > >
Re:[DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect
Thanks for the KIP Chris, adding this check makes total sense. I do have one question. The second paragraph in the Public Interfaces section states: "If the connector generated excessive tasks after being reconfigured, then any existing tasks for the connector will be allowed to continue running, unless that existing set of tasks also exceeds the tasks.max property." Would not failing the connector land us in the second scenario of 'Rejected Alternatives'? From: dev@kafka.apache.org At: 11/11/23 00:27:44 UTC-5:00To: dev@kafka.apache.org Subject: [DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect Hi all, I'd like to open up KIP-1004 for discussion: https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+ property+in+Kafka+Connect As a brief summary: this KIP proposes that the Kafka Connect runtime start failing connectors that generate a greater number of tasks than the tasks.max property, with an optional emergency override that can be used to continue running these (probably-buggy) connectors if absolutely necessary. I'll be taking time off most of the next three weeks, so response latency may be a bit higher than usual, but I wanted to kick off the discussion in case we can land this in time for the upcoming 3.7.0 release. Cheers, Chris
[jira] [Created] (KAFKA-15841) Add Support for Topic-Level Partitioning in Kafka Connect
Henrique Mota created KAFKA-15841: - Summary: Add Support for Topic-Level Partitioning in Kafka Connect Key: KAFKA-15841 URL: https://issues.apache.org/jira/browse/KAFKA-15841 Project: Kafka Issue Type: Improvement Components: connect Reporter: Henrique Mota In our organization, we utilize JDBC sink connectors to consume data from various topics, where each topic is dedicated to a specific tenant with a single partition. Recently, we developed a custom sink based on the standard JDBC sink, enabling us to pause consumption of a topic when encountering problematic records. However, we face limitations within Kafka Connect, as it doesn't allow for appropriate partitioning of topics among workers. We attempted a workaround by breaking down the topics list within the 'topics' parameter. Unfortunately, Kafka Connect overrides this parameter after invoking the {{taskConfigs(int maxTasks)}} method from the {{org.apache.kafka.connect.connector.Connector}} class. We request the addition of support in Kafka Connect to enable the partitioning of topics among workers without requiring a fork. This enhancement would facilitate better load distribution and allow for more flexible configurations, particularly in scenarios where topics are dedicated to different tenants. -- This message was sent by Atlassian Jira (v8.20.10#820010)
[DISCUSS] KIP-1004: Enforce tasks.max property in Kafka Connect
Hi all, I'd like to open up KIP-1004 for discussion: https://cwiki.apache.org/confluence/display/KAFKA/KIP-1004%3A+Enforce+tasks.max+property+in+Kafka+Connect As a brief summary: this KIP proposes that the Kafka Connect runtime start failing connectors that generate a greater number of tasks than the tasks.max property, with an optional emergency override that can be used to continue running these (probably-buggy) connectors if absolutely necessary. I'll be taking time off most of the next three weeks, so response latency may be a bit higher than usual, but I wanted to kick off the discussion in case we can land this in time for the upcoming 3.7.0 release. Cheers, Chris
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Chris, Regarding item 4: Thanks for clarification. I really missed it. I've updated the 'compatibility' section [1] and prototype [2] accordingly. Regarding item 5: Perfect naming is the one of hardest things and not my strong point. > The best I've been able to come up with is establishing > two new interfaces: ClientSslEngineFactory and ServerSslEngineFactory I cannot catch an idea with two interfaces because this interfaces make sense only for private code, but user always must implements both especially for Connect config. > Maybe BaseSslEngineFactory? Let me know what you think. All I can think of is: - BaseSslEngineFactory; - StaticSslEngineFactory; - NonReconfigurableSslEngineFactory. But I'm afraid that it would be hard to find sponsors to review this. My proposal: restrict the scope of this KIP to Kafka Connect config and file a new KIP to refactor the `SslEngineFactory`. I see that there are KIPs with smaller changes. I guess simple and clear short steps make success. [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer#KIP967:SupportcustomSSLconfigurationforKafkaConnectRestServer-Compatibility,Deprecation,andMigrationPlan [2]. https://github.com/apache/kafka/pull/14203/files#diff-2c8e6c18e6cc9082d6b8eb38d1ee00e7c8bd4819b5b32e68e7d1fce2f7570c47R195 -- With best regards, Taras Ledkov On Thu, Nov 2, 2023 at 8:25 PM Chris Egerton wrote: > > Hi Taras, > > Thanks for the changes to the KIP! > > Regarding item 4: I think some background may be helpful for people without > context on the Connect code base. The current parsing logic for SSL-related > properties used with the REST API is to use all worker properties prefixed > with "listeners.https." (ignoring admin listeners here for the sake of > discussion, but the logic is essentially the same for them albeit with a > different prefix), or if no properties are found with that prefix, all > worker properties that are defined in the ConfigDef for the worker (which > includes all properties provided by SslConfigs::addClientSupport [1]). This > logic comes from usage of the AbstractConfig::valuesWithPrefixAllOrNothing > method [2] by the SSLUtils class [3], [4]. > > It seems like the restriction on only using properties defined by the > worker is a potential problem here, since an SSL engine factory may want to > use property names that aren't available out-of-the-box with Kafka Connect > (i.e., which aren't defined in the DistributedConfig [5] or > StandaloneConfig [6] class). > > It also sounds like (reading the KIP again) we're proposing that SSL engine > factory classes only be configured with properties prefixed with > "listeners.https.", without the fallback on all non-prefixed worker > properties if none are found. Wouldn't this be a breaking, > backwards-incompatible change for existing Connect clusters that are > configured to use SSL with top-level properties instead of ones prefixed > with "listeners.https."? > > IMO the most reasonable compromise here would be to keep almost the exact > same logic for finding SSL-related properties for the REST API, but without > the restriction on predefined configs for the fallback. So it'd be: if any > properties are present with the "listeners.https." prefix, pass only those > to the SSL engine factory, and otherwise, pass all non-prefixed worker > properties to the factory (without checking to see if they're defined by > the worker already). Do you agree? If so, we should clarify this in the KIP. > > > Regarding item 5: This is a little hairy. I'd prefer to avoid leaving > optional methods in the interface since they'll just add FUD for anyone who > doesn't find the right corner of our docs to read, and they may also imply > that the Connect runtime supports functionality that it doesn't (i.e., > dynamic reconfiguration of SSL). But I do agree that it'd be convenient to > support existing implementations of the SslEngineFactory interface, so a > completely separate interface isn't good here either. I like the idea of > introducing a superinterface for SslEngineFactory, but I've been struggling > to think of a good name for it. The best I've been able to come up with is > establishing two new interfaces: ClientSslEngineFactory and > ServerSslEngineFactory, which contain SSLEngine > createClientSslEngine(String peerHost, int peerPort, String > endpointIdentification) and SSLEngine createServerSslEngine(String > peerHost, int peerPort), respectively. We could require custom Connect SSL > engine factories to implement both ClientSslEngineFactory > and ServerSslEngineFactory. But frankly, the only reason two interfaces > seem clearer than one here is that I can't think of a better name than > SslEngin
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Taras, Thanks for the changes to the KIP! Regarding item 4: I think some background may be helpful for people without context on the Connect code base. The current parsing logic for SSL-related properties used with the REST API is to use all worker properties prefixed with "listeners.https." (ignoring admin listeners here for the sake of discussion, but the logic is essentially the same for them albeit with a different prefix), or if no properties are found with that prefix, all worker properties that are defined in the ConfigDef for the worker (which includes all properties provided by SslConfigs::addClientSupport [1]). This logic comes from usage of the AbstractConfig::valuesWithPrefixAllOrNothing method [2] by the SSLUtils class [3], [4]. It seems like the restriction on only using properties defined by the worker is a potential problem here, since an SSL engine factory may want to use property names that aren't available out-of-the-box with Kafka Connect (i.e., which aren't defined in the DistributedConfig [5] or StandaloneConfig [6] class). It also sounds like (reading the KIP again) we're proposing that SSL engine factory classes only be configured with properties prefixed with "listeners.https.", without the fallback on all non-prefixed worker properties if none are found. Wouldn't this be a breaking, backwards-incompatible change for existing Connect clusters that are configured to use SSL with top-level properties instead of ones prefixed with "listeners.https."? IMO the most reasonable compromise here would be to keep almost the exact same logic for finding SSL-related properties for the REST API, but without the restriction on predefined configs for the fallback. So it'd be: if any properties are present with the "listeners.https." prefix, pass only those to the SSL engine factory, and otherwise, pass all non-prefixed worker properties to the factory (without checking to see if they're defined by the worker already). Do you agree? If so, we should clarify this in the KIP. Regarding item 5: This is a little hairy. I'd prefer to avoid leaving optional methods in the interface since they'll just add FUD for anyone who doesn't find the right corner of our docs to read, and they may also imply that the Connect runtime supports functionality that it doesn't (i.e., dynamic reconfiguration of SSL). But I do agree that it'd be convenient to support existing implementations of the SslEngineFactory interface, so a completely separate interface isn't good here either. I like the idea of introducing a superinterface for SslEngineFactory, but I've been struggling to think of a good name for it. The best I've been able to come up with is establishing two new interfaces: ClientSslEngineFactory and ServerSslEngineFactory, which contain SSLEngine createClientSslEngine(String peerHost, int peerPort, String endpointIdentification) and SSLEngine createServerSslEngine(String peerHost, int peerPort), respectively. We could require custom Connect SSL engine factories to implement both ClientSslEngineFactory and ServerSslEngineFactory. But frankly, the only reason two interfaces seem clearer than one here is that I can't think of a better name than SslEngineFactory to capture the two methods we need. Maybe BaseSslEngineFactory? Let me know what you think. [1] - https://github.com/apache/kafka/blob/57662efec9207446ffb32db179e4adf6bff76e18/clients/src/main/java/org/apache/kafka/common/config/SslConfigs.java#L138-L158 [2] - https://github.com/apache/kafka/blob/57662efec9207446ffb32db179e4adf6bff76e18/clients/src/main/java/org/apache/kafka/common/config/AbstractConfig.java#L317-L340 [3] - https://github.com/apache/kafka/blob/57662efec9207446ffb32db179e4adf6bff76e18/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/rest/util/SSLUtils.java#L44 [4] - https://github.com/apache/kafka/blob/57662efec9207446ffb32db179e4adf6bff76e18/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/rest/util/SSLUtils.java#L67C70-L67C82 [5] - https://github.com/apache/kafka/blob/57662efec9207446ffb32db179e4adf6bff76e18/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/distributed/DistributedConfig.java [6] - https://github.com/apache/kafka/blob/57662efec9207446ffb32db179e4adf6bff76e18/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/standalone/StandaloneConfig.java Cheers, Chris On Thu, Nov 2, 2023 at 10:44 AM Taras Ledkov wrote: > Hi Chris, > > Thanks a lot for such a close review. > > > 1. The "ssl.engine.factory.class" property was originally added for Kafka > > brokers in KIP-519 [1]. It'd be nice to link to that KIP (possibly in a > > "Background" section? > Added "Background" section. > > > 2. Can we clarify that the new "listeners.https.ssl.engine.factory.class" > > property (and the way that the engine factory is configured with all > >
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Chris, Thanks a lot for such a close review. > 1. The "ssl.engine.factory.class" property was originally added for Kafka > brokers in KIP-519 [1]. It'd be nice to link to that KIP (possibly in a > "Background" section? Added "Background" section. > 2. Can we clarify that the new "listeners.https.ssl.engine.factory.class" > property (and the way that the engine factory is configured with all > properties prefixed with "listeners.https.") will also affect MirrorMaker 2 Great observation! I've skipped it. Added to section "Compatibility, Deprecation, and Migration Plan" > 3. We don't need to specify in the KIP that the > org.apache.kafka.connect.runtime.rest.util.SSLUtils class will be removed, Dropped. > 4. The test plan 4.1. Dropped "Default SSL behavior and compatibility" 4.2. As far as i see `RestForwardingIntegrationTest` contains the bug that is annihilated by the strange behavior of the `AbstractConfig#valuesWithPrefixAllOrNothing` at the `SSLUtils#createServerSideSslContextFactory` (or invalid usage of this method in this place). All ssl properties are not prefixed with "listeners.https.". I guess that the definitions of SSL properties at the root of the `WorkerConfig` is unclean / buggy / may lead to unexpected behavior. So, I would like to fix / improve this test. > 5. There are several methods in the SslEngineFactory interface that don't > seem applicable for Kafka Connect. Cool idea. Do you have any ideas about design? My proposal: - Extract base interface from `SslEngineFactory` and simple refactoring of the `SslFactory`; or - Do nothing and document the fact that implementation of these methods are not necessary for the SSL Engine Factory used for Connect / MM2. But I guess, a user that implements its own SslEngineFactory` for a the broker just prefers to reuse the code as is. [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer#KIP967:SupportcustomSSLconfigurationforKafkaConnectRestServer-Background -- With best regards, Taras Ledkov On Mon, Oct 30, 2023 at 8:07 PM Chris Egerton wrote: > > Hi Taras, > > Thanks for the KIP! I have some feedback but ultimately I like this > proposal: > > 1. The "ssl.engine.factory.class" property was originally added for Kafka > brokers in KIP-519 [1]. It'd be nice to link to that KIP (possibly in a > "Background" section?) so that reviewers who don't have that context can > find it quickly without having to dig through commit histories in the code > base. > > 2. Can we clarify that the new "listeners.https.ssl.engine.factory.class" > property (and the way that the engine factory is configured with all > properties prefixed with "listeners.https.") will also affect MirrorMaker 2 > clusters with the internal REST server introduced by KIP-710 [2] enabled? > > 3. We don't need to specify in the KIP that the > org.apache.kafka.connect.runtime.rest.util.SSLUtils class will be removed, > since that class isn't part of public API (i.e., nobody should be using > that class directly in external projects). If you're ever in doubt about > which classes are part of the public API for the project, you can check the > Javadocs [3]; if it's part of our public API, it should be included in > them. The same applies for changes to the > org.apache.kafka.common.security.ssl.SslFactory class. > > 4. The test plan includes an integration test for "Default SSL behavior and > compatibility"--is this necessary? Doesn't the > existing org.apache.kafka.connect.integration.RestForwardingIntegrationTest > give us sufficient coverage already? Similarly, the test plan includes an > integration test for "RestClient creation" and calls out > the RestForwardingIntegrationTest--don't we already create RestClient > instances in that test (like here [4])? It seems like this part of the KIP > may implicitly include tests that are already covered by the existing code > base, but if that's the case, it'd be nice to see this clarified as the > assumption is usually that items in the test plan cover changes that will > have to be implemented for the KIP. > > 5. There are several methods in the SslEngineFactory interface that don't > seem applicable for Kafka Connect (or MM2): shouldBeRebuilt(Map Object> nextConfigs), reconfigurableConfigs(), and possibly keystore() and > truststore(). Does it make sense to require users to implement these? It > seems like a new interface may make more sense here. > > [1] - > https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=128650952 > [2] - > https://cwiki.apache.org/confluence/display/KAFKA/KIP-710%3A+Full+support+for+distributed+mode+in+dedicated+M
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Taras, Thanks for the KIP! I have some feedback but ultimately I like this proposal: 1. The "ssl.engine.factory.class" property was originally added for Kafka brokers in KIP-519 [1]. It'd be nice to link to that KIP (possibly in a "Background" section?) so that reviewers who don't have that context can find it quickly without having to dig through commit histories in the code base. 2. Can we clarify that the new "listeners.https.ssl.engine.factory.class" property (and the way that the engine factory is configured with all properties prefixed with "listeners.https.") will also affect MirrorMaker 2 clusters with the internal REST server introduced by KIP-710 [2] enabled? 3. We don't need to specify in the KIP that the org.apache.kafka.connect.runtime.rest.util.SSLUtils class will be removed, since that class isn't part of public API (i.e., nobody should be using that class directly in external projects). If you're ever in doubt about which classes are part of the public API for the project, you can check the Javadocs [3]; if it's part of our public API, it should be included in them. The same applies for changes to the org.apache.kafka.common.security.ssl.SslFactory class. 4. The test plan includes an integration test for "Default SSL behavior and compatibility"--is this necessary? Doesn't the existing org.apache.kafka.connect.integration.RestForwardingIntegrationTest give us sufficient coverage already? Similarly, the test plan includes an integration test for "RestClient creation" and calls out the RestForwardingIntegrationTest--don't we already create RestClient instances in that test (like here [4])? It seems like this part of the KIP may implicitly include tests that are already covered by the existing code base, but if that's the case, it'd be nice to see this clarified as the assumption is usually that items in the test plan cover changes that will have to be implemented for the KIP. 5. There are several methods in the SslEngineFactory interface that don't seem applicable for Kafka Connect (or MM2): shouldBeRebuilt(Map nextConfigs), reconfigurableConfigs(), and possibly keystore() and truststore(). Does it make sense to require users to implement these? It seems like a new interface may make more sense here. [1] - https://cwiki.apache.org/confluence/pages/viewpage.action?pageId=128650952 [2] - https://cwiki.apache.org/confluence/display/KAFKA/KIP-710%3A+Full+support+for+distributed+mode+in+dedicated+MirrorMaker+2.0+clusters [3] - https://kafka.apache.org/36/javadoc/index.html?overview-summary.html [4] - https://github.com/apache/kafka/blob/9dbee599f13997effd8f7e278fd7256b850c8813/connect/runtime/src/test/java/org/apache/kafka/connect/integration/RestForwardingIntegrationTest.java#L161 Cheers, Chris On Thu, Oct 12, 2023 at 7:40 AM Taras Ledkov wrote: > Hi Ashwin, > > > I was referring to (and did not understand) the removal of L141 in > clients/src/main/java/org/apache/kafka/common/security/ssl/SslFactory.java > This line is moved to "new" private method `instantiateSslEngineFactory0 > `. Please take a look at the `SslFactory:L132` at the patch. > Just dummy refactoring. > > > Yes, I think this class [SslEngineFactory] should be moved to something > like `server-common` module - but would like any of the committers to > comment on this. > Sorry, not catch an idea. > SslEngineFactory - public interface is placed at the 'clients' project. I > don't know a more common place >
Re: [VOTE] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
I support this KIP +1 > 23 окт. 2023 г., в 20:20, Greg Harris > написал(а): > > Hey Taras, > > Thanks for the KIP! > > The design you propose follows the conventions started in KIP-519, and > should feel natural to operators familiar with the broker feature. > I also like that we're able to clean up some connect-specific > functionality and make the codebase more consistent. > > +1 (binding) > > Thanks, > Greg > > On Fri, Oct 20, 2023 at 8:03 AM Taras Ledkov wrote: >> >> Hi Kafka Team. >> >> II'd like to call a vote on KIP-967: Support custom SSL configuration for >> Kafka Connect RestServer [1]. >> Discussion thread [2] was started more then 2 month ago and there was not >> any negative or critical comments. >> >> [1]. >> https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer >> [2]. https://lists.apache.org/thread/w0vmbf1yzgjo7hkzyyzjjnb509x6s9qq >> >> -- >> With best regards, >> Taras Ledkov
Re: [VOTE] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hey Taras, Thanks for the KIP! The design you propose follows the conventions started in KIP-519, and should feel natural to operators familiar with the broker feature. I also like that we're able to clean up some connect-specific functionality and make the codebase more consistent. +1 (binding) Thanks, Greg On Fri, Oct 20, 2023 at 8:03 AM Taras Ledkov wrote: > > Hi Kafka Team. > > II'd like to call a vote on KIP-967: Support custom SSL configuration for > Kafka Connect RestServer [1]. > Discussion thread [2] was started more then 2 month ago and there was not any > negative or critical comments. > > [1]. > https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer > [2]. https://lists.apache.org/thread/w0vmbf1yzgjo7hkzyyzjjnb509x6s9qq > > -- > With best regards, > Taras Ledkov
[VOTE] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Kafka Team. II'd like to call a vote on KIP-967: Support custom SSL configuration for Kafka Connect RestServer [1]. Discussion thread [2] was started more then 2 month ago and there was not any negative or critical comments. [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer [2]. https://lists.apache.org/thread/w0vmbf1yzgjo7hkzyyzjjnb509x6s9qq -- With best regards, Taras Ledkov
Re:[DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Ashwin, > I was referring to (and did not understand) the removal of L141 in > clients/src/main/java/org/apache/kafka/common/security/ssl/SslFactory.java This line is moved to "new" private method `instantiateSslEngineFactory0 `. Please take a look at the `SslFactory:L132` at the patch. Just dummy refactoring. > Yes, I think this class [SslEngineFactory] should be moved to something like > `server-common` module - but would like any of the committers to comment on > this. Sorry, not catch an idea. SslEngineFactory - public interface is placed at the 'clients' project. I don't know a more common place
Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi all, Thanks for the votes! I'll cast a final +1 myself and close the vote out. This KIP passes with the following +1 votes (and no +0 or -1 votes): • Greg Harris (binding) • Yash Mayya (binding) • Federico Valeri • Mickael Maison (binding) • hudeqi • Hector Geraldino • Chris Egerton (binding, author) I've begun implementing the feature and plan on publishing the first PR sometime this week or the next. Cheers, Chris On Mon, Oct 9, 2023 at 2:32 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) < hgerald...@bloomberg.net> wrote: > Good stuff, +1 (non-binding) from me as well > > De: dev@kafka.apache.org A: 10/09/23 05:16:06 UTC-4:00A: > dev@kafka.apache.org > Subject: Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka > Connect > > Hi Chris, > > +1 (non binding) > > Thanks > Fede > > On Sun, Oct 8, 2023 at 10:11 AM Yash Mayya wrote: > > > > Hi Chris, > > > > Thanks for the KIP! > > +1 (binding) > > > > Yash > > > > On Fri, Oct 6, 2023 at 9:54 PM Greg Harris > > > wrote: > > > > > Hey Chris, > > > > > > Thanks for the KIP! > > > I think that preserving the ephemeral nature of the logging change is > > > the right choice here, and using the config topic for intra-cluster > > > broadcast is better than REST forwarding. > > > > > > +1 (binding) > > > > > > Thanks, > > > Greg > > > > > > On Fri, Oct 6, 2023 at 9:05 AM Chris Egerton > > > wrote: > > > > > > > > Hi all, > > > > > > > > I'd like to call for a vote on KIP-976, which augments the existing > > > dynamic > > > > logger adjustment REST API for Kafka Connect to apply changes > > > cluster-wide > > > > instead on a per-worker basis. > > > > > > > > The KIP: > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976:+Cluster-wide+dynamic+ > log+adjustment+for+Kafka+Connect > > > > > > > > The discussion thread: > > > > https://lists.apache.org/thread/w3x3f3jmyd1vfjxho06y8xgt6mhhzpl5 > > > > > > > > Cheers, > > > > > > > > Chris > > > > > >
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hello Taras, > Do you think that something needs to be corrected in KIP to make it more understandable without PR? Do you have any advice? Ha - no. I just wanted to thank you for sharing the PR which helped me as a newbie. > If I understood the question correctly: I was referring to (and did not understand) the removal of L141 in clients/src/main/java/org/apache/kafka/common/security/ssl/SslFactory.java in https://github.com/apache/kafka/pull/14203/files. But you are right - this can be discussed in the final PR. > There might be a better place for this public static method that creates the SslEngineFactory. Yes, I think this class should be moved to something like `server-common` module - but would like any of the committers to comment on this. Thanks, Ashwin On Fri, Sep 29, 2023 at 9:22 PM Taras Ledkov wrote: > Hi Ashwin, > > Thanks a lot for your review. > > > Thanks for the KIP and the PR (which helped me understand the change). > Do you think that something needs to be corrected in KIP to make it more > understandable without PR? Do you have any advice? > > > I could not understand one thing though - In > https://github.com/apache/kafka/pull/14203/, > > why did you have to remove the code which sets sslEngineFactoryConfig in > instantiateSslEngineFactory? > If I understood the question correctly: > I've refactored this method a bit. > SslFactory#instantiateSslEngineFactory was a private not-static method. > I've separated the code that really creates new instance of the > SslEngineFactory and place it into a public static method. There might be a > better place for this public static method that creates the > SslEngineFactory. I think we will discuss this at the final PR. Current PR > is just a demo / prototype to play. >
Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Good stuff, +1 (non-binding) from me as well De: dev@kafka.apache.org A: 10/09/23 05:16:06 UTC-4:00A: dev@kafka.apache.org Subject: Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect Hi Chris, +1 (non binding) Thanks Fede On Sun, Oct 8, 2023 at 10:11 AM Yash Mayya wrote: > > Hi Chris, > > Thanks for the KIP! > +1 (binding) > > Yash > > On Fri, Oct 6, 2023 at 9:54 PM Greg Harris > wrote: > > > Hey Chris, > > > > Thanks for the KIP! > > I think that preserving the ephemeral nature of the logging change is > > the right choice here, and using the config topic for intra-cluster > > broadcast is better than REST forwarding. > > > > +1 (binding) > > > > Thanks, > > Greg > > > > On Fri, Oct 6, 2023 at 9:05 AM Chris Egerton > > wrote: > > > > > > Hi all, > > > > > > I'd like to call for a vote on KIP-976, which augments the existing > > dynamic > > > logger adjustment REST API for Kafka Connect to apply changes > > cluster-wide > > > instead on a per-worker basis. > > > > > > The KIP: > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976:+Cluster-wide+dynamic+ log+adjustment+for+Kafka+Connect > > > > > > The discussion thread: > > > https://lists.apache.org/thread/w3x3f3jmyd1vfjxho06y8xgt6mhhzpl5 > > > > > > Cheers, > > > > > > Chris > >
Re: Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, +1 (non-binding) Finally, there is no need to use external intrusion tools to change the log level of kafka connect online! Thanks for the KIP! best, hudeqi
Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, +1 (binding) Thanks for the KIP! On Mon, Oct 9, 2023 at 11:16 AM Federico Valeri wrote: > > Hi Chris, > > +1 (non binding) > > Thanks > Fede > > On Sun, Oct 8, 2023 at 10:11 AM Yash Mayya wrote: > > > > Hi Chris, > > > > Thanks for the KIP! > > +1 (binding) > > > > Yash > > > > On Fri, Oct 6, 2023 at 9:54 PM Greg Harris > > wrote: > > > > > Hey Chris, > > > > > > Thanks for the KIP! > > > I think that preserving the ephemeral nature of the logging change is > > > the right choice here, and using the config topic for intra-cluster > > > broadcast is better than REST forwarding. > > > > > > +1 (binding) > > > > > > Thanks, > > > Greg > > > > > > On Fri, Oct 6, 2023 at 9:05 AM Chris Egerton > > > wrote: > > > > > > > > Hi all, > > > > > > > > I'd like to call for a vote on KIP-976, which augments the existing > > > dynamic > > > > logger adjustment REST API for Kafka Connect to apply changes > > > cluster-wide > > > > instead on a per-worker basis. > > > > > > > > The KIP: > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976:+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > > > > > The discussion thread: > > > > https://lists.apache.org/thread/w3x3f3jmyd1vfjxho06y8xgt6mhhzpl5 > > > > > > > > Cheers, > > > > > > > > Chris > > >
Re:[DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi, Kafka team. 1. Ping to review KIP. 2. I dare say that the low activity in the discussion of KIP-967 means that KIP-967 is ready for voting? -- With best regards Taras Ledkov
Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, +1 (non binding) Thanks Fede On Sun, Oct 8, 2023 at 10:11 AM Yash Mayya wrote: > > Hi Chris, > > Thanks for the KIP! > +1 (binding) > > Yash > > On Fri, Oct 6, 2023 at 9:54 PM Greg Harris > wrote: > > > Hey Chris, > > > > Thanks for the KIP! > > I think that preserving the ephemeral nature of the logging change is > > the right choice here, and using the config topic for intra-cluster > > broadcast is better than REST forwarding. > > > > +1 (binding) > > > > Thanks, > > Greg > > > > On Fri, Oct 6, 2023 at 9:05 AM Chris Egerton > > wrote: > > > > > > Hi all, > > > > > > I'd like to call for a vote on KIP-976, which augments the existing > > dynamic > > > logger adjustment REST API for Kafka Connect to apply changes > > cluster-wide > > > instead on a per-worker basis. > > > > > > The KIP: > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976:+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > > > The discussion thread: > > > https://lists.apache.org/thread/w3x3f3jmyd1vfjxho06y8xgt6mhhzpl5 > > > > > > Cheers, > > > > > > Chris > >
Re: Kafka Connect - Customize REST request headers
Thank you for the explanation, Chris. In case it helps, what I'm looking for is similar to KIP 577[1]. My specific example involves a hard-coded key/value pair that needs to be used for pod-to-pod as I can connect to any worker without that specific header, but workers cannot communicate among themselves without it. To clarify, my environment is behind Istio[2], where Egress Traffic can be created using the following format: `..svc.cluster.local`. For example, a request among workers should be: curl -H "Host: ..svc.cluster.local" workerIP:PORT Regarding temporary solutions, I've explored options like utilizing a proxy but I am running within containers that can complicate it further, along with the possibilities of patching, recompiling, or replacing the connect-runtime jar temporarily. I think that something like this might work but I need to test it : private static void addHeadersToRequest(HttpHeaders headers, Request req) { req.header("Host","..svc.cluster.local"); if (headers != null) { String credentialAuthorization = headers.getHeaderString(HttpHeaders.AUTHORIZATION); if (credentialAuthorization != null) { req.header(HttpHeaders.AUTHORIZATION, credentialAuthorization); } } } This is of course risky and it would be significantly more convenient if this functionality is integrated into Kafka Connect itself [1] https://cwiki.apache.org/confluence/display/KAFKA/KIP+577%3A+Allow+HTTP+Response+Headers+to+be+Configured+for+Kafka+Connect [2] https://istio.io/ On Sat, 07 Oct 2023 02:05:14 -0400 Chris Egerton wrote --- Hi Yeikel, Neat question! And thanks for the link to the RestClient code; very helpful. I don't believe there's a way to configure Kafka Connect to add these headers to forwarded requests right now. You may be able to do some kind of out-of-band proxy magic to intercept forwarded requests and insert the proper headers there. I don't see a reason for Kafka Connect to only forward authorization headers, even after examining the PR [1] and corresponding Jira ticket [2] that altered the RestClient class to begin including authorization headers in forwarded REST requests. We may be able to tweak the RestClient to include all headers instead of just the authorization header. I know that this doesn't help your immediate situation, but if other committers and contributors agree that the change would be beneficial, we may be able to include it in the next release (which may be 3.7.0, or a patch release for 3.4, 3.5, or 3.6). Alternatively, we may have to gate such a change behind a feature flag (either a coarse-grained boolean that enables/disables forwarding of all non-authorization headers, or more fine-grained logic such as include/exclude lists or even regexes), which would require a KIP and may take longer to release. I've CC'd the dev list to gather their perspective on this potential change, and to solicit their input on possible workarounds that may be useful to you sooner than the next release takes place. [1] - https://github.com/apache/kafka/pull/6791 [2] - https://issues.apache.org/jira/browse/KAFKA-8404 Cheers, Chris On Fri, Oct 6, 2023 at 10:14 PM Yeikel Santana <mailto:em...@yeikel.com> wrote: > Hello everyone, > > I'm currently running Kafka Connect behind a firewall that mandates the > inclusion of a specific header. This situation becomes particularly > challenging when forwarding requests among multiple workers, as it appears > that only the Authorization header is included in the request. > > I'm wondering if there's a way to customize the headers of Kafka Connect > before they are forwarded between workers. From my observations, it seems > that this capability may not be available[1], and only the response headers > can be customized. > > I'd appreciate any realistic alternatives or suggestions you may have in > mind. > > Thanks! > > > > > > > [1] > https://github.com/apache/kafka/blob/trunk/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/rest/RestClient.java#L191-L198
Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, Thanks for the KIP! +1 (binding) Yash On Fri, Oct 6, 2023 at 9:54 PM Greg Harris wrote: > Hey Chris, > > Thanks for the KIP! > I think that preserving the ephemeral nature of the logging change is > the right choice here, and using the config topic for intra-cluster > broadcast is better than REST forwarding. > > +1 (binding) > > Thanks, > Greg > > On Fri, Oct 6, 2023 at 9:05 AM Chris Egerton > wrote: > > > > Hi all, > > > > I'd like to call for a vote on KIP-976, which augments the existing > dynamic > > logger adjustment REST API for Kafka Connect to apply changes > cluster-wide > > instead on a per-worker basis. > > > > The KIP: > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976:+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > The discussion thread: > > https://lists.apache.org/thread/w3x3f3jmyd1vfjxho06y8xgt6mhhzpl5 > > > > Cheers, > > > > Chris >
Re: Kafka Connect - Customize REST request headers
Hi Yeikel, Neat question! And thanks for the link to the RestClient code; very helpful. I don't believe there's a way to configure Kafka Connect to add these headers to forwarded requests right now. You may be able to do some kind of out-of-band proxy magic to intercept forwarded requests and insert the proper headers there? I don't see a reason for Kafka Connect to only forward authorization headers, even after examining the PR [1] and corresponding Jira ticket [2] that altered the RestClient class to begin including authorization headers in forwarded REST requests. We may be able to tweak the RestClient to include all headers instead of just the authorization header. I know that this doesn't help your immediate situation, but if other committers and contributors agree that the change would be beneficial, we may be able to include it in the next release (which may be 3.7.0, or a patch release for 3.4, 3.5, or 3.6). Alternatively, we may have to gate such a change behind a feature flag (either a coarse-grained boolean that enables/disables forwarding of all non-authorization headers, or more fine-grained logic such as include/exclude lists or even regexes), which would require a KIP and may take longer to release. I've CC'd the dev list to gather their perspective on this potential change, and to solicit their input on possible workarounds that may be useful to you sooner than the next release takes place. [1] - https://github.com/apache/kafka/pull/6791 [2] - https://issues.apache.org/jira/browse/KAFKA-8404 Cheers, Chris On Fri, Oct 6, 2023 at 10:14 PM Yeikel Santana wrote: > Hello everyone, > > I'm currently running Kafka Connect behind a firewall that mandates the > inclusion of a specific header. This situation becomes particularly > challenging when forwarding requests among multiple workers, as it appears > that only the Authorization header is included in the request. > > I'm wondering if there's a way to customize the headers of Kafka Connect > before they are forwarded between workers. From my observations, it seems > that this capability may not be available[1], and only the response headers > can be customized. > > I'd appreciate any realistic alternatives or suggestions you may have in > mind. > > Thanks! > > > > > > > [1] > https://github.com/apache/kafka/blob/trunk/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/rest/RestClient.java#L191-L198
Re: [VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hey Chris, Thanks for the KIP! I think that preserving the ephemeral nature of the logging change is the right choice here, and using the config topic for intra-cluster broadcast is better than REST forwarding. +1 (binding) Thanks, Greg On Fri, Oct 6, 2023 at 9:05 AM Chris Egerton wrote: > > Hi all, > > I'd like to call for a vote on KIP-976, which augments the existing dynamic > logger adjustment REST API for Kafka Connect to apply changes cluster-wide > instead on a per-worker basis. > > The KIP: > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976:+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > The discussion thread: > https://lists.apache.org/thread/w3x3f3jmyd1vfjxho06y8xgt6mhhzpl5 > > Cheers, > > Chris
[VOTE] KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi all, I'd like to call for a vote on KIP-976, which augments the existing dynamic logger adjustment REST API for Kafka Connect to apply changes cluster-wide instead on a per-worker basis. The KIP: https://cwiki.apache.org/confluence/display/KAFKA/KIP-976:+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect The discussion thread: https://lists.apache.org/thread/w3x3f3jmyd1vfjxho06y8xgt6mhhzpl5 Cheers, Chris
Re:[DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Ashwin, Thanks a lot for your review. > Thanks for the KIP and the PR (which helped me understand the change). Do you think that something needs to be corrected in KIP to make it more understandable without PR? Do you have any advice? > I could not understand one thing though - In > https://github.com/apache/kafka/pull/14203/, > why did you have to remove the code which sets sslEngineFactoryConfig in > instantiateSslEngineFactory? If I understood the question correctly: I've refactored this method a bit. SslFactory#instantiateSslEngineFactory was a private not-static method. I've separated the code that really creates new instance of the SslEngineFactory and place it into a public static method. There might be a better place for this public static method that creates the SslEngineFactory. I think we will discuss this at the final PR. Current PR is just a demo / prototype to play.
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Taras, Thanks for the KIP and the PR (which helped me understand the change). This is a useful feature, change is small and reuses existing functionality in clients/../SslFactory.java - so hopefully, this KIP will get accepted. I could not understand one thing though - In https://github.com/apache/kafka/pull/14203/, why did you have to remove the code which sets sslEngineFactoryConfig in instantiateSslEngineFactory ? Thanks, Ashwin On Tue, Sep 26, 2023 at 6:39 PM Taras Ledkov wrote: > Hi Kafka Team. > > Ping... >
Re:[DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Kafka Team. Ping...
[GitHub] [kafka-site] mimaison merged pull request #552: Add Kafka Connect book
mimaison merged PR #552: URL: https://github.com/apache/kafka-site/pull/552 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: dev-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka-site] mimaison opened a new pull request, #552: Add Kafka Connect book
mimaison opened a new pull request, #552: URL: https://github.com/apache/kafka-site/pull/552 (no comment) -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: dev-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
Re:[DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Kafka Team. Looks like the code freeze of 3.6.0 release done. I hope that community members have more time for review. Please pay your attention for the KIP-967: Support custom SSL configuration for Kafka Connect RestServer [1]. The purpose of this KIP is add ability to use custom SSL factory to configure Kafka Connect RestServer. It looks like the interface 'SslEngineFactory' may be used with simple adapters. The prototype of the patch is available on PR#14203 [2]. It is not a final/clean patch yet. Just for demo & discuss. Thanks in advance for leaving a review! [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer [2]. https://github.com/apache/kafka/pull/14203
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
> request. Time will tell... > > > Thanks again, all! > > Cheers, > > Chris > > On Wed, Sep 6, 2023 at 8:36 AM Yash Mayya wrote: > > > Hi Chris, > > > > Thanks for the clarification on the last modified timestamp tracking here > > and on the KIP, things look good to me now. > > > > On the persistence front, I hadn't considered the interplay between > levels > > set by the log level REST APIs and those set by the log4j configuration > > files, and the user confusion that could arise from it. I think your > > argument for keeping the changes made by these log level admin endpoints > > ephemeral makes sense, thanks! > > > > Hi Sagar, > > > > > The entire namespace name in the config > > > records key (along with logger-cluster prefix) > > > seems to be a bit too verbose. My first > > > thought was to not have the prefix > > > org.apache.kafka.connect in the keys > > > considering it is the root namespace. But > > > since logging can be enabled at a root level, > > > can we just use initials like (o.a.k.c) which is > > > also a standard practice. > > > > We allow modifying the log levels for any namespace - i.e. even packages > > and classes outside of Kafka Connect itself (think connector classes, > > dependencies etc.). I'm not sure I follow how we'd avoid naming clashes > > without using the fully qualified name? > > > > > I was also thinking if we could introduce a > > > new parameter which takes a subset of > > > worker ids and enables logging for them in > > > one go > > > > The future section already talks about potentially introducing new scopes > > such as a specific connector or a specific connector plugin. Are there > any > > use cases apart from these that would be satisfied by allowing changing > the > > log levels on a subset of workers in a Kafka Connect cluster? > > > > Thanks, > > Yash > > > > On Wed, Sep 6, 2023 at 7:41 AM Sagar wrote: > > > > > Hey Chris, > > > > > > Thanks for making the updates. > > > > > > The updated definition of last_modified looks good to me. As a > > continuation > > > to point number 2, could we also mention that this could be used to get > > > insights into the propagation of the cluster wide log level updates. It > > is > > > implicit but probably better to add it I feel. > > > > > > Regarding > > > > > > It's a little indirect on the front of worker restart behavior, so let > me > > > > know if that especially should be fleshed out a bit more (possibly by > > > > calling it out in the "Config topic records" section). > > > > > > > > > Yeah I would lean on the side of calling it out explicitly. Since the > > > behaviour is similar to the current dynamically set log levels (i.e > > > resetting to the log4j config files levels) so we can call it out > stating > > > that similarity just for completeness sake. It would be useful info for > > > new/medium level users reading the KIP considering worker restarts is > not > > > uncommon. > > > > > > > > > Thanks, I'm glad that this seems reasonable. I've tentatively added > this > > to > > > > the rejected alternatives section, but am happy to continue the > > > > conversation if anyone wants to explore that possibility further. > > > > > > > > > +1 > > > > > > I had a nit level suggestion but not sure if it makes sense but would > > still > > > call it out. The entire namespace name in the config records key (along > > > with logger-cluster prefix) seems to be a bit too verbose. My first > > thought > > > was to not have the prefix org.apache.kafka.connect in the keys > > considering > > > it is the root namespace. But since logging can be enabled at a root > > level, > > > can we just use initials like (o.a.k.c) which is also a standard > > practice. > > > Let me know what you think? > > > > > > Lastly, I was also thinking if we could introduce a new parameter which > > > takes a subset of worker ids and enables logging for them in one go. > But > > > this is already achievable by invoking scope=worker endpoint n times to > > > reflect on n workers so maybe not a necessary change. But this could be > > > useful on a large cluster. Do you think this is worth listing in the > > Future > > > Work sectio
[GitHub] [kafka-site] C0urante merged pull request #539: KAFKA-14876: Add stopped state to Kafka Connect Administration docs section
C0urante merged PR #539: URL: https://github.com/apache/kafka-site/pull/539 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: dev-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi all, Thanks again for the reviews! Sagar: > The updated definition of last_modified looks good to me. As a continuation to point number 2, could we also mention that this could be used to get insights into the propagation of the cluster wide log level updates. It is implicit but probably better to add it I feel. Sure, done. Added to the end of the "Config topic records" section: "There may be some delay between when a REST request with scope=cluster is received and when all workers have read the corresponding record from the config topic. The last modified timestamp (details above) can serve as a rudimentary tool to provide insight into the propagation of a cluster-wide log level adjustment." > Yeah I would lean on the side of calling it out explicitly. Since the behaviour is similar to the current dynamically set log levels (i.e resetting to the log4j config files levels) so we can call it out stating that similarity just for completeness sake. It would be useful info for new/medium level users reading the KIP considering worker restarts is not uncommon. Alright, did this too. Added near the end of the "Config topic records" section: "Restarting a worker will cause it to discard all cluster-wide dynamic log level adjustments, and revert to the levels specified in its Log4j configuration. This mirrors the current behavior with per-worker dynamic log level adjustments." > I had a nit level suggestion but not sure if it makes sense but would still call it out. The entire namespace name in the config records key (along with logger-cluster prefix) seems to be a bit too verbose. My first thought was to not have the prefix org.apache.kafka.connect in the keys considering it is the root namespace. But since logging can be enabled at a root level, can we just use initials like (o.a.k.c) which is also a standard practice. Let me know what you think? Considering these records aren't meant to be user-visible, there doesn't seem to be a pressing need to reduce their key sizes (though I'll happily admit that to human eyes, the format is a bit ugly). IMO the increase in implementation complexity isn't quite worth it, especially considering there are plenty of logging namespaces that won't begin with "org.apache.kafka.connect" (likely including all third-party connector code), like Yash mentions. Is there a motivation for this suggestion that I'm missing? > Lastly, I was also thinking if we could introduce a new parameter which takes a subset of worker ids and enables logging for them in one go. But this is already achievable by invoking scope=worker endpoint n times to reflect on n workers so maybe not a necessary change. But this could be useful on a large cluster. Do you think this is worth listing in the Future Work section? It's not important so can be ignored as well. Hmmm... I think I'd rather leave this out for now because I'm just not certain enough it'd be useful. The one advantage I can think of is targeting specific workers that are behind a load balancer, but being able to identify those workers would be a challenge in that scenario anyways. Besides that, are there any cases that couldn't be addressed more holistically by targeting based on connector/connector type, like Yash asks? Ashwin: Glad we're on the same page RE request forwarding and integration vs. system tests! Let me know if anything else comes up that you'd like to discuss. Yash: Glad that it makes sense to keep these changes ephemeral. I'm not quite confident enough to put persistent updates in the "Future work" section but have a sneaking suspicion that this isn't the last we'll see of that request. Time will tell... Thanks again, all! Cheers, Chris On Wed, Sep 6, 2023 at 8:36 AM Yash Mayya wrote: > Hi Chris, > > Thanks for the clarification on the last modified timestamp tracking here > and on the KIP, things look good to me now. > > On the persistence front, I hadn't considered the interplay between levels > set by the log level REST APIs and those set by the log4j configuration > files, and the user confusion that could arise from it. I think your > argument for keeping the changes made by these log level admin endpoints > ephemeral makes sense, thanks! > > Hi Sagar, > > > The entire namespace name in the config > > records key (along with logger-cluster prefix) > > seems to be a bit too verbose. My first > > thought was to not have the prefix > > org.apache.kafka.connect in the keys > > considering it is the root namespace. But > > since logging can be enabled at a root level, > > can we just use initials like (o.a.k.c) which is > > also a standard practice. > > We allow modifying the log levels for any namespace - i.e. even packages > and classes outside of Kafka Connect itself (think connector classes, > dependencies etc.). I'm no
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, Thanks for the clarification on the last modified timestamp tracking here and on the KIP, things look good to me now. On the persistence front, I hadn't considered the interplay between levels set by the log level REST APIs and those set by the log4j configuration files, and the user confusion that could arise from it. I think your argument for keeping the changes made by these log level admin endpoints ephemeral makes sense, thanks! Hi Sagar, > The entire namespace name in the config > records key (along with logger-cluster prefix) > seems to be a bit too verbose. My first > thought was to not have the prefix > org.apache.kafka.connect in the keys > considering it is the root namespace. But > since logging can be enabled at a root level, > can we just use initials like (o.a.k.c) which is > also a standard practice. We allow modifying the log levels for any namespace - i.e. even packages and classes outside of Kafka Connect itself (think connector classes, dependencies etc.). I'm not sure I follow how we'd avoid naming clashes without using the fully qualified name? > I was also thinking if we could introduce a > new parameter which takes a subset of > worker ids and enables logging for them in > one go The future section already talks about potentially introducing new scopes such as a specific connector or a specific connector plugin. Are there any use cases apart from these that would be satisfied by allowing changing the log levels on a subset of workers in a Kafka Connect cluster? Thanks, Yash On Wed, Sep 6, 2023 at 7:41 AM Sagar wrote: > Hey Chris, > > Thanks for making the updates. > > The updated definition of last_modified looks good to me. As a continuation > to point number 2, could we also mention that this could be used to get > insights into the propagation of the cluster wide log level updates. It is > implicit but probably better to add it I feel. > > Regarding > > It's a little indirect on the front of worker restart behavior, so let me > > know if that especially should be fleshed out a bit more (possibly by > > calling it out in the "Config topic records" section). > > > Yeah I would lean on the side of calling it out explicitly. Since the > behaviour is similar to the current dynamically set log levels (i.e > resetting to the log4j config files levels) so we can call it out stating > that similarity just for completeness sake. It would be useful info for > new/medium level users reading the KIP considering worker restarts is not > uncommon. > > > Thanks, I'm glad that this seems reasonable. I've tentatively added this to > > the rejected alternatives section, but am happy to continue the > > conversation if anyone wants to explore that possibility further. > > > +1 > > I had a nit level suggestion but not sure if it makes sense but would still > call it out. The entire namespace name in the config records key (along > with logger-cluster prefix) seems to be a bit too verbose. My first thought > was to not have the prefix org.apache.kafka.connect in the keys considering > it is the root namespace. But since logging can be enabled at a root level, > can we just use initials like (o.a.k.c) which is also a standard practice. > Let me know what you think? > > Lastly, I was also thinking if we could introduce a new parameter which > takes a subset of worker ids and enables logging for them in one go. But > this is already achievable by invoking scope=worker endpoint n times to > reflect on n workers so maybe not a necessary change. But this could be > useful on a large cluster. Do you think this is worth listing in the Future > Work section? It's not important so can be ignored as well. > > Thanks! > Sagar. > > > On Wed, Sep 6, 2023 at 12:08 AM Chris Egerton > wrote: > > > Hi Sagar, > > > > Thanks for your thoughts! Responses inline: > > > > > 1) Considering that you have now clarified here that last_modified > field > > would be stored in-memory, it is not mentioned in the KIP. Although > that's > > something that's understandable, it wasn't apparent when reading the KIP. > > Probably, we could mention it? Also, what happens if a worker restarts? > In > > that case, since the log level update message would be pre-dating the > > startup of the worker, it would be ignored? We should probably mention > that > > behaviour as well IMO. > > > > I've tweaked the second paragraph of the "Last modified timestamp" > section > > to try to clarify this without getting too verbose: "Modification times > > will be tracked in-memory and determined by when they are applied by the > > worker, as opposed to when they are requested by the user
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
a 404 response" > > > I'm curious to know what the rationale here is? In KIP-745, the stated > reasoning behind ignoring restart requests during worker startup was that > the worker will anyway be starting all connectors and tasks assigned to it > so the restart request is essentially meaningless. With the log level API > however, wouldn't it make more sense to apply any cluster scoped > modifications to new workers in the cluster too? This also applies to any > workers that are restarted soon after a request is made to *PUT > /admin/loggers/{logger}?scope=cluster *on another worker. Maybe we could > stack up all the cluster scoped log level modification requests during the > config topic read at worker startup and apply the latest ones per namespace > (assuming older records haven't already been compacted) after we've > finished reading to the end of the config topic? > > It's definitely possible to implement persistent cluster-wide logging level > adjustments, with the possible exception that none will be applied until > the worker has finished reading to the end of the config topic. The > rationale for keeping these updates ephemeral is to continue to give > priority to workers' Log4j configuration files, with the underlying > philosophy that this endpoint is still only intended for debugging > purposes, as opposed to cluster-wide configuration. Permanent changes can > already be made by tweaking the Log4j file for a worker and then restarting > it. If a restart is too expensive for a permanent change, then the change > can be applied immediately via the REST API, and staged via the Log4j > configuration file (which will then be used the next time the worker is > restarted, whenever that happens). > > We could potentially explore persistent cluster-wide logging adjustments in > the future (or, if people feel strongly, right now), but it'd be more > complex. How would we handle the case where a user has updated their Log4j > configuration file but continues to see the same logging level for a > namespace, in a way that greatly reduces or even eliminates the odds of > headaches or footguns? How would we allow users to reset or undo > cluster-wide configuration updates, in order to revert back to whatever's > specified in their Log4j configuration file? Would we want to give users > control over whether a dynamic update is persistent or ephemeral? What > would the implications be for finer-grained scoping (especially with > regards to resetting dynamic changes)? Overall it doesn't seem worth the > work to tackle all of this right now, as long as we're not painting > ourselves into a corner. But as always, happy to hear what you and others > think! > > > Federico: > > > Due to the idempotent nature of PUT, I guess that the last_modified > timestamp won't change if the same request is repeated successively. > Should we add a unit test for that? > > Great call! I've added this to the testing plan with unit and end-to-end > tests (seems fairly cheap to do both; LMK if e2e feels like overkill > though). > > > Thanks again for all of your thoughts! > > Cheers, > > Chris > > On Mon, Sep 4, 2023 at 2:53 AM Federico Valeri > wrote: > > > Hi Chris, thanks. This looks like a useful feature. > > > > Due to the idempotent nature of PUT, I guess that the last_modified > > timestamp won't change if the same request is repeated successively. > > Should we add a unit test for that? > > > > On Mon, Sep 4, 2023 at 6:17 AM Ashwin > > wrote: > > > > > > Hi Chris, > > > > > > Thanks for thinking about this useful feature ! > > > I had a question regarding > > > > > > > Since cluster metadata is not required to handle these types of > > request, > > > they will not be forwarded to the leader > > > > > > And later, we also mention about supporting more scope types in the > > future. > > > Don't you foresee a future scope type which may require cluster > metadata > > ? > > > In that case, isn't it better to forward the requests to the leader in > > the > > > initial implementation ? > > > > > > I would also recommend an additional system test for Standalone herder > to > > > ensure that the new scope parameter is honored and the response > contains > > > the last modified time. > > > > > > Thanks, > > > Ashwin > > > > > > On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton > > > wrote: > > > > > > > Hi all, > > > > > > > > Can't imagine a worse time to publish a new KIP (it's late on a > Friday > >
[GitHub] [kafka-site] yashmayya opened a new pull request, #539: KAFKA-14876: Add stopped state to Kafka Connect Administration docs section
yashmayya opened a new pull request, #539: URL: https://github.com/apache/kafka-site/pull/539 - See https://github.com/apache/kafka/pull/14336 and https://github.com/apache/kafka/commit/b8cf3e31747f7193024c36f3381c0dd5bd22158c -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: dev-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
> > > > Cheers, > > > > > > Chris > > > > > > On Mon, Sep 4, 2023 at 2:53 AM Federico Valeri > > > wrote: > > > > > > > Hi Chris, thanks. This looks like a useful feature. > > > > > > > > Due to the idempotent nature of PUT, I guess that the last_modified > > > > timestamp won't change if the same request is repeated successively. > > > > Should we add a unit test for that? > > > > > > > > On Mon, Sep 4, 2023 at 6:17 AM Ashwin > > > > wrote: > > > > > > > > > > Hi Chris, > > > > > > > > > > Thanks for thinking about this useful feature ! > > > > > I had a question regarding > > > > > > > > > > > Since cluster metadata is not required to handle these types of > > > > request, > > > > > they will not be forwarded to the leader > > > > > > > > > > And later, we also mention about supporting more scope types in the > > > > future. > > > > > Don't you foresee a future scope type which may require cluster > > > metadata > > > > ? > > > > > In that case, isn't it better to forward the requests to the leader > > in > > > > the > > > > > initial implementation ? > > > > > > > > > > I would also recommend an additional system test for Standalone > > herder > > > to > > > > > ensure that the new scope parameter is honored and the response > > > contains > > > > > the last modified time. > > > > > > > > > > Thanks, > > > > > Ashwin > > > > > > > > > > On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton > > > > > > > > wrote: > > > > > > > > > > > Hi all, > > > > > > > > > > > > Can't imagine a worse time to publish a new KIP (it's late on a > > > Friday > > > > and > > > > > > we're in the middle of the 3.6.0 release), but I wanted to put > > forth > > > > > > KIP-976 for discussion: > > > > > > > > > > > > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > > > > > > > > > TL;DR: The API to dynamically adjust log levels at runtime with > > > > Connect is > > > > > > great, and I'd like to augment it with support to adjust log > levels > > > for > > > > > > every worker in the cluster (instead of just the worker that > > receives > > > > the > > > > > > REST request). > > > > > > > > > > > > I look forward to everyone's thoughts, but expect that this will > > > > probably > > > > > > take a bump or two once the dust has settled on 3.6.0. Huge > thanks > > to > > > > > > everyone that's contributed to that release so far, especially > our > > > > release > > > > > > manager Satish! > > > > > > > > > > > > Cheers, > > > > > > > > > > > > Chris > > > > > > > > > > > > > > > >
[GitHub] [kafka-site] C0urante merged pull request #538: MINOR: Add missing entries for Kafka Connect to the documentation's table of contents
C0urante merged PR #538: URL: https://github.com/apache/kafka-site/pull/538 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: dev-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
re persistent cluster-wide logging adjustments > in > > the future (or, if people feel strongly, right now), but it'd be more > > complex. How would we handle the case where a user has updated their > Log4j > > configuration file but continues to see the same logging level for a > > namespace, in a way that greatly reduces or even eliminates the odds of > > headaches or footguns? How would we allow users to reset or undo > > cluster-wide configuration updates, in order to revert back to whatever's > > specified in their Log4j configuration file? Would we want to give users > > control over whether a dynamic update is persistent or ephemeral? What > > would the implications be for finer-grained scoping (especially with > > regards to resetting dynamic changes)? Overall it doesn't seem worth the > > work to tackle all of this right now, as long as we're not painting > > ourselves into a corner. But as always, happy to hear what you and others > > think! > > > > > > Federico: > > > > > Due to the idempotent nature of PUT, I guess that the last_modified > > timestamp won't change if the same request is repeated successively. > > Should we add a unit test for that? > > > > Great call! I've added this to the testing plan with unit and end-to-end > > tests (seems fairly cheap to do both; LMK if e2e feels like overkill > > though). > > > > > > Thanks again for all of your thoughts! > > > > Cheers, > > > > Chris > > > > On Mon, Sep 4, 2023 at 2:53 AM Federico Valeri > > wrote: > > > > > Hi Chris, thanks. This looks like a useful feature. > > > > > > Due to the idempotent nature of PUT, I guess that the last_modified > > > timestamp won't change if the same request is repeated successively. > > > Should we add a unit test for that? > > > > > > On Mon, Sep 4, 2023 at 6:17 AM Ashwin > > > wrote: > > > > > > > > Hi Chris, > > > > > > > > Thanks for thinking about this useful feature ! > > > > I had a question regarding > > > > > > > > > Since cluster metadata is not required to handle these types of > > > request, > > > > they will not be forwarded to the leader > > > > > > > > And later, we also mention about supporting more scope types in the > > > future. > > > > Don't you foresee a future scope type which may require cluster > > metadata > > > ? > > > > In that case, isn't it better to forward the requests to the leader > in > > > the > > > > initial implementation ? > > > > > > > > I would also recommend an additional system test for Standalone > herder > > to > > > > ensure that the new scope parameter is honored and the response > > contains > > > > the last modified time. > > > > > > > > Thanks, > > > > Ashwin > > > > > > > > On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton > > > > > wrote: > > > > > > > > > Hi all, > > > > > > > > > > Can't imagine a worse time to publish a new KIP (it's late on a > > Friday > > > and > > > > > we're in the middle of the 3.6.0 release), but I wanted to put > forth > > > > > KIP-976 for discussion: > > > > > > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > > > > > > > TL;DR: The API to dynamically adjust log levels at runtime with > > > Connect is > > > > > great, and I'd like to augment it with support to adjust log levels > > for > > > > > every worker in the cluster (instead of just the worker that > receives > > > the > > > > > REST request). > > > > > > > > > > I look forward to everyone's thoughts, but expect that this will > > > probably > > > > > take a bump or two once the dust has settled on 3.6.0. Huge thanks > to > > > > > everyone that's contributed to that release so far, especially our > > > release > > > > > manager Satish! > > > > > > > > > > Cheers, > > > > > > > > > > Chris > > > > > > > > > > >
[GitHub] [kafka-site] yashmayya commented on pull request #538: MINOR: Add missing entries for Kafka Connect to the documentation's table of contents
yashmayya commented on PR #538: URL: https://github.com/apache/kafka-site/pull/538#issuecomment-1707109599 cc @C0urante -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: dev-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
[GitHub] [kafka-site] yashmayya opened a new pull request, #538: MINOR: Add missing entries for Kafka Connect to the documentation's table of contents
yashmayya opened a new pull request, #538: URL: https://github.com/apache/kafka-site/pull/538 - https://github.com/apache/kafka/pull/14337 - Some of Kafka Connect's top level headings (``) and sub top level headings (``) in the documentation weren't added to the documentation's table of contents. This patch rectifies that. Before: https://github.com/apache/kafka/assets/23502577/7a0d6425-05d0-4ebc-b62f-6495e300aa27;> After: https://github.com/apache/kafka/assets/23502577/f0f71e02-06c2-4ea1-9d65-376e09f9cd6f;> -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: dev-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
the restart request is essentially meaningless. With the log level API > however, wouldn't it make more sense to apply any cluster scoped > modifications to new workers in the cluster too? This also applies to any > workers that are restarted soon after a request is made to *PUT > /admin/loggers/{logger}?scope=cluster *on another worker. Maybe we could > stack up all the cluster scoped log level modification requests during the > config topic read at worker startup and apply the latest ones per namespace > (assuming older records haven't already been compacted) after we've > finished reading to the end of the config topic? > > It's definitely possible to implement persistent cluster-wide logging level > adjustments, with the possible exception that none will be applied until > the worker has finished reading to the end of the config topic. The > rationale for keeping these updates ephemeral is to continue to give > priority to workers' Log4j configuration files, with the underlying > philosophy that this endpoint is still only intended for debugging > purposes, as opposed to cluster-wide configuration. Permanent changes can > already be made by tweaking the Log4j file for a worker and then restarting > it. If a restart is too expensive for a permanent change, then the change > can be applied immediately via the REST API, and staged via the Log4j > configuration file (which will then be used the next time the worker is > restarted, whenever that happens). > > We could potentially explore persistent cluster-wide logging adjustments in > the future (or, if people feel strongly, right now), but it'd be more > complex. How would we handle the case where a user has updated their Log4j > configuration file but continues to see the same logging level for a > namespace, in a way that greatly reduces or even eliminates the odds of > headaches or footguns? How would we allow users to reset or undo > cluster-wide configuration updates, in order to revert back to whatever's > specified in their Log4j configuration file? Would we want to give users > control over whether a dynamic update is persistent or ephemeral? What > would the implications be for finer-grained scoping (especially with > regards to resetting dynamic changes)? Overall it doesn't seem worth the > work to tackle all of this right now, as long as we're not painting > ourselves into a corner. But as always, happy to hear what you and others > think! > > > Federico: > > > Due to the idempotent nature of PUT, I guess that the last_modified > timestamp won't change if the same request is repeated successively. > Should we add a unit test for that? > > Great call! I've added this to the testing plan with unit and end-to-end > tests (seems fairly cheap to do both; LMK if e2e feels like overkill > though). > > > Thanks again for all of your thoughts! > > Cheers, > > Chris > > On Mon, Sep 4, 2023 at 2:53 AM Federico Valeri > wrote: > > > Hi Chris, thanks. This looks like a useful feature. > > > > Due to the idempotent nature of PUT, I guess that the last_modified > > timestamp won't change if the same request is repeated successively. > > Should we add a unit test for that? > > > > On Mon, Sep 4, 2023 at 6:17 AM Ashwin > > wrote: > > > > > > Hi Chris, > > > > > > Thanks for thinking about this useful feature ! > > > I had a question regarding > > > > > > > Since cluster metadata is not required to handle these types of > > request, > > > they will not be forwarded to the leader > > > > > > And later, we also mention about supporting more scope types in the > > future. > > > Don't you foresee a future scope type which may require cluster > metadata > > ? > > > In that case, isn't it better to forward the requests to the leader in > > the > > > initial implementation ? > > > > > > I would also recommend an additional system test for Standalone herder > to > > > ensure that the new scope parameter is honored and the response > contains > > > the last modified time. > > > > > > Thanks, > > > Ashwin > > > > > > On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton > > > wrote: > > > > > > > Hi all, > > > > > > > > Can't imagine a worse time to publish a new KIP (it's late on a > Friday > > and > > > > we're in the middle of the 3.6.0 release), but I wanted to put forth > > > > KIP-976 for discussion: > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > > > > > TL;DR: The API to dynamically adjust log levels at runtime with > > Connect is > > > > great, and I'd like to augment it with support to adjust log levels > for > > > > every worker in the cluster (instead of just the worker that receives > > the > > > > REST request). > > > > > > > > I look forward to everyone's thoughts, but expect that this will > > probably > > > > take a bump or two once the dust has settled on 3.6.0. Huge thanks to > > > > everyone that's contributed to that release so far, especially our > > release > > > > manager Satish! > > > > > > > > Cheers, > > > > > > > > Chris > > > > > > >
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
nt cluster-wide logging adjustments in the future (or, if people feel strongly, right now), but it'd be more complex. How would we handle the case where a user has updated their Log4j configuration file but continues to see the same logging level for a namespace, in a way that greatly reduces or even eliminates the odds of headaches or footguns? How would we allow users to reset or undo cluster-wide configuration updates, in order to revert back to whatever's specified in their Log4j configuration file? Would we want to give users control over whether a dynamic update is persistent or ephemeral? What would the implications be for finer-grained scoping (especially with regards to resetting dynamic changes)? Overall it doesn't seem worth the work to tackle all of this right now, as long as we're not painting ourselves into a corner. But as always, happy to hear what you and others think! Federico: > Due to the idempotent nature of PUT, I guess that the last_modified timestamp won't change if the same request is repeated successively. Should we add a unit test for that? Great call! I've added this to the testing plan with unit and end-to-end tests (seems fairly cheap to do both; LMK if e2e feels like overkill though). Thanks again for all of your thoughts! Cheers, Chris On Mon, Sep 4, 2023 at 2:53 AM Federico Valeri wrote: > Hi Chris, thanks. This looks like a useful feature. > > Due to the idempotent nature of PUT, I guess that the last_modified > timestamp won't change if the same request is repeated successively. > Should we add a unit test for that? > > On Mon, Sep 4, 2023 at 6:17 AM Ashwin > wrote: > > > > Hi Chris, > > > > Thanks for thinking about this useful feature ! > > I had a question regarding > > > > > Since cluster metadata is not required to handle these types of > request, > > they will not be forwarded to the leader > > > > And later, we also mention about supporting more scope types in the > future. > > Don't you foresee a future scope type which may require cluster metadata > ? > > In that case, isn't it better to forward the requests to the leader in > the > > initial implementation ? > > > > I would also recommend an additional system test for Standalone herder to > > ensure that the new scope parameter is honored and the response contains > > the last modified time. > > > > Thanks, > > Ashwin > > > > On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton > > wrote: > > > > > Hi all, > > > > > > Can't imagine a worse time to publish a new KIP (it's late on a Friday > and > > > we're in the middle of the 3.6.0 release), but I wanted to put forth > > > KIP-976 for discussion: > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > > > TL;DR: The API to dynamically adjust log levels at runtime with > Connect is > > > great, and I'd like to augment it with support to adjust log levels for > > > every worker in the cluster (instead of just the worker that receives > the > > > REST request). > > > > > > I look forward to everyone's thoughts, but expect that this will > probably > > > take a bump or two once the dust has settled on 3.6.0. Huge thanks to > > > everyone that's contributed to that release so far, especially our > release > > > manager Satish! > > > > > > Cheers, > > > > > > Chris > > > >
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, thanks. This looks like a useful feature. Due to the idempotent nature of PUT, I guess that the last_modified timestamp won't change if the same request is repeated successively. Should we add a unit test for that? On Mon, Sep 4, 2023 at 6:17 AM Ashwin wrote: > > Hi Chris, > > Thanks for thinking about this useful feature ! > I had a question regarding > > > Since cluster metadata is not required to handle these types of request, > they will not be forwarded to the leader > > And later, we also mention about supporting more scope types in the future. > Don't you foresee a future scope type which may require cluster metadata ? > In that case, isn't it better to forward the requests to the leader in the > initial implementation ? > > I would also recommend an additional system test for Standalone herder to > ensure that the new scope parameter is honored and the response contains > the last modified time. > > Thanks, > Ashwin > > On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton > wrote: > > > Hi all, > > > > Can't imagine a worse time to publish a new KIP (it's late on a Friday and > > we're in the middle of the 3.6.0 release), but I wanted to put forth > > KIP-976 for discussion: > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > > > TL;DR: The API to dynamically adjust log levels at runtime with Connect is > > great, and I'd like to augment it with support to adjust log levels for > > every worker in the cluster (instead of just the worker that receives the > > REST request). > > > > I look forward to everyone's thoughts, but expect that this will probably > > take a bump or two once the dust has settled on 3.6.0. Huge thanks to > > everyone that's contributed to that release so far, especially our release > > manager Satish! > > > > Cheers, > > > > Chris > >
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
> If no modifications to a logging namespace have > been made, won't the namespace itself be omitted > from the response? It looks like we currently only > return loggers that have non-null log levels in the *> **GET /admin/loggers* endpoint. This can be ignored - I didn't account for the fact that at worker startup we'll still have loggers with non-null log levels - the root logger and any other named loggers which have explicit log levels configured in the log4j properties. On Mon, Sep 4, 2023 at 12:00 PM Yash Mayya wrote: > Hi Chris, > > Thanks for the KIP, this looks like a really useful addition to Kafka > Connect's log level REST APIs! I have a few questions and comments: > > > If no modifications to the namespace have > > been made since the worker was started, > > they will be null > > If no modifications to a logging namespace have been made, won't the > namespace itself be omitted from the response? It looks like we currently > only return loggers that have non-null log levels in the *GET > /admin/loggers* endpoint. > > > Last modified timestamp > > From the proposed changes section, it isn't very clear to me how we'll be > tracking this last modified timestamp to be returned in responses for the *GET > /admin/loggers* and *GET /admin/loggers/{logger}* endpoints. Could you > please elaborate on this? Also, will we track the last modified timestamp > even for worker scoped modifications where we won't write any records to > the config topic and the requests will essentially be processed > synchronously? > > > Record values will have the following format, where ${level} is the new > logging level for the namespace: > > In the current synchronous implementation for the *PUT > /admin/loggers/{logger} *endpoint, we return a 404 error if the level is > invalid (i.e. not one of TRACE, DEBUG, WARN etc.). Since the new cluster > scoped variant of the endpoint will be asynchronous, can we also add a > validation to synchronously surface erroneous log levels to users? > > > Workers that have not yet completed startup > > will ignore these records, including if the worker > > reads one during the read-to-end of the config > > topic that all workers perform during startup. > > I'm curious to know what the rationale here is? In KIP-745, the stated > reasoning behind ignoring restart requests during worker startup was that > the worker will anyway be starting all connectors and tasks assigned to it > so the restart request is essentially meaningless. With the log level API > however, wouldn't it make more sense to apply any cluster scoped > modifications to new workers in the cluster too? This also applies to any > workers that are restarted soon after a request is made to *PUT > /admin/loggers/{logger}?scope=cluster *on another worker. Maybe we could > stack up all the cluster scoped log level modification requests during the > config topic read at worker startup and apply the latest ones per namespace > (assuming older records haven't already been compacted) after we've > finished reading to the end of the config topic? > > > if you're debugging the distributed herder, you > > need all the help you can get > > > > As an aside, thanks for the impressively thorough testing plan in the KIP! > > > Hi Ashwin, > > > isn't it better to forward the requests to the > > leader in the initial implementation ? > > Would there be any downside to only introducing leader forwarding for > connector/task specific scope types in the future (rather than introducing > it at the outset here where it isn't strictly required)? > > > I would also recommend an additional system > > test for Standalone herder to ensure that the > > new scope parameter is honored and the response > > contains the last modified time. > > Can't this be sufficiently covered using unit and / or integration tests? > System tests are fairly expensive to run in terms of overall test runtime > and they are also not run on every PR or commit to trunk / feature branches > (unlike unit tests and integration tests). > > Thanks, > Yash > > On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton > wrote: > >> Hi all, >> >> Can't imagine a worse time to publish a new KIP (it's late on a Friday and >> we're in the middle of the 3.6.0 release), but I wanted to put forth >> KIP-976 for discussion: >> >> https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect >> >> TL;DR: The API to dynamically adjust log levels at runtime with Connect is >> great, and I'd like to augment it with support to adjust log levels for >> every worker in the cluster (instead of just the worker that receives the >> REST request). >> >> I look forward to everyone's thoughts, but expect that this will probably >> take a bump or two once the dust has settled on 3.6.0. Huge thanks to >> everyone that's contributed to that release so far, especially our release >> manager Satish! >> >> Cheers, >> >> Chris >> >
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, Thanks for the KIP, this looks like a really useful addition to Kafka Connect's log level REST APIs! I have a few questions and comments: > If no modifications to the namespace have > been made since the worker was started, > they will be null If no modifications to a logging namespace have been made, won't the namespace itself be omitted from the response? It looks like we currently only return loggers that have non-null log levels in the *GET /admin/loggers* endpoint. > Last modified timestamp >From the proposed changes section, it isn't very clear to me how we'll be tracking this last modified timestamp to be returned in responses for the *GET /admin/loggers* and *GET /admin/loggers/{logger}* endpoints. Could you please elaborate on this? Also, will we track the last modified timestamp even for worker scoped modifications where we won't write any records to the config topic and the requests will essentially be processed synchronously? > Record values will have the following format, where ${level} is the new logging level for the namespace: In the current synchronous implementation for the *PUT /admin/loggers/{logger} *endpoint, we return a 404 error if the level is invalid (i.e. not one of TRACE, DEBUG, WARN etc.). Since the new cluster scoped variant of the endpoint will be asynchronous, can we also add a validation to synchronously surface erroneous log levels to users? > Workers that have not yet completed startup > will ignore these records, including if the worker > reads one during the read-to-end of the config > topic that all workers perform during startup. I'm curious to know what the rationale here is? In KIP-745, the stated reasoning behind ignoring restart requests during worker startup was that the worker will anyway be starting all connectors and tasks assigned to it so the restart request is essentially meaningless. With the log level API however, wouldn't it make more sense to apply any cluster scoped modifications to new workers in the cluster too? This also applies to any workers that are restarted soon after a request is made to *PUT /admin/loggers/{logger}?scope=cluster *on another worker. Maybe we could stack up all the cluster scoped log level modification requests during the config topic read at worker startup and apply the latest ones per namespace (assuming older records haven't already been compacted) after we've finished reading to the end of the config topic? > if you're debugging the distributed herder, you > need all the help you can get As an aside, thanks for the impressively thorough testing plan in the KIP! Hi Ashwin, > isn't it better to forward the requests to the > leader in the initial implementation ? Would there be any downside to only introducing leader forwarding for connector/task specific scope types in the future (rather than introducing it at the outset here where it isn't strictly required)? > I would also recommend an additional system > test for Standalone herder to ensure that the > new scope parameter is honored and the response > contains the last modified time. Can't this be sufficiently covered using unit and / or integration tests? System tests are fairly expensive to run in terms of overall test runtime and they are also not run on every PR or commit to trunk / feature branches (unlike unit tests and integration tests). Thanks, Yash On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton wrote: > Hi all, > > Can't imagine a worse time to publish a new KIP (it's late on a Friday and > we're in the middle of the 3.6.0 release), but I wanted to put forth > KIP-976 for discussion: > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > TL;DR: The API to dynamically adjust log levels at runtime with Connect is > great, and I'd like to augment it with support to adjust log levels for > every worker in the cluster (instead of just the worker that receives the > REST request). > > I look forward to everyone's thoughts, but expect that this will probably > take a bump or two once the dust has settled on 3.6.0. Huge thanks to > everyone that's contributed to that release so far, especially our release > manager Satish! > > Cheers, > > Chris >
Re: KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi Chris, Thanks for thinking about this useful feature ! I had a question regarding > Since cluster metadata is not required to handle these types of request, they will not be forwarded to the leader And later, we also mention about supporting more scope types in the future. Don't you foresee a future scope type which may require cluster metadata ? In that case, isn't it better to forward the requests to the leader in the initial implementation ? I would also recommend an additional system test for Standalone herder to ensure that the new scope parameter is honored and the response contains the last modified time. Thanks, Ashwin On Sat, Sep 2, 2023 at 5:12 AM Chris Egerton wrote: > Hi all, > > Can't imagine a worse time to publish a new KIP (it's late on a Friday and > we're in the middle of the 3.6.0 release), but I wanted to put forth > KIP-976 for discussion: > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect > > TL;DR: The API to dynamically adjust log levels at runtime with Connect is > great, and I'd like to augment it with support to adjust log levels for > every worker in the cluster (instead of just the worker that receives the > REST request). > > I look forward to everyone's thoughts, but expect that this will probably > take a bump or two once the dust has settled on 3.6.0. Huge thanks to > everyone that's contributed to that release so far, especially our release > manager Satish! > > Cheers, > > Chris >
KIP-976: Cluster-wide dynamic log adjustment for Kafka Connect
Hi all, Can't imagine a worse time to publish a new KIP (it's late on a Friday and we're in the middle of the 3.6.0 release), but I wanted to put forth KIP-976 for discussion: https://cwiki.apache.org/confluence/display/KAFKA/KIP-976%3A+Cluster-wide+dynamic+log+adjustment+for+Kafka+Connect TL;DR: The API to dynamically adjust log levels at runtime with Connect is great, and I'd like to augment it with support to adjust log levels for every worker in the cluster (instead of just the worker that receives the REST request). I look forward to everyone's thoughts, but expect that this will probably take a bump or two once the dust has settled on 3.6.0. Huge thanks to everyone that's contributed to that release so far, especially our release manager Satish! Cheers, Chris
[jira] [Created] (KAFKA-15408) Restart failed tasks in Kafka Connect up to a configurable max-tries
Patrick Pang created KAFKA-15408: Summary: Restart failed tasks in Kafka Connect up to a configurable max-tries Key: KAFKA-15408 URL: https://issues.apache.org/jira/browse/KAFKA-15408 Project: Kafka Issue Type: New Feature Components: KafkaConnect Reporter: Patrick Pang h2. Issue Currently, Kafka Connect just reports failed tasks on REST API, with the error. Users are expected to monitor the status and restart individual connectors if there is transient errors. Unfortunately these are common for database connectors, e.g. transient connection error, flip of DNS, database downtime, etc. Kafka Connect silently failing due to these scenarios would lead to stale data downstream. h2. Proposal Kafka Connect should be able to restart failed tasks automatically, up to a configurable max-tries. -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hello, Taras. I found this KIP useful. We already has an ability to setup custom SslEngineFactory via ‘ssl.engine.factory.class' So it’s looks logical to extend this feature to connect rest. AFAIK many organization adopts custom SSL storage like HashiCorp Vault or similar so native integration will be useful > 14 авг. 2023 г., в 12:42, Taras Ledkov написал(а): > > Hi Kafka Team. > > I would like to start a discussion for KIP-967: Support custom SSL > configuration for Kafka Connect RestServer [1]. > The purpose of this KIP is add ability to use custom SSL factory to configure > Kafka Connect RestServer. > It looks like the interface 'SslEngineFactory' may be used with simple > adapters. > > The prototype of the patch is available on PR#14203 [2]. > It is not a final/clean patch yet. Just for demo & discuss. > > Thanks in advance for leaving a review! > > [1]. > https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer > [2]. https://github.com/apache/kafka/pull/14203 > > -- > With best regards, > Taras Ledkov
[DISCUSS] KIP-967: Support custom SSL configuration for Kafka Connect RestServer
Hi Kafka Team. I would like to start a discussion for KIP-967: Support custom SSL configuration for Kafka Connect RestServer [1]. The purpose of this KIP is add ability to use custom SSL factory to configure Kafka Connect RestServer. It looks like the interface 'SslEngineFactory' may be used with simple adapters. The prototype of the patch is available on PR#14203 [2]. It is not a final/clean patch yet. Just for demo & discuss. Thanks in advance for leaving a review! [1]. https://cwiki.apache.org/confluence/display/KAFKA/KIP-967%3A+Support+custom+SSL+configuration+for+Kafka+Connect+RestServer [2]. https://github.com/apache/kafka/pull/14203 -- With best regards, Taras Ledkov
[jira] [Created] (KAFKA-15335) Support custom SSL configuration for Kafka Connect RestServer
Taras Ledkov created KAFKA-15335: Summary: Support custom SSL configuration for Kafka Connect RestServer Key: KAFKA-15335 URL: https://issues.apache.org/jira/browse/KAFKA-15335 Project: Kafka Issue Type: New Feature Components: connect Reporter: Taras Ledkov Root to track KIP-967 -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect
Thanks Mickael! The KIP has passed with 3 binding votes (Chris Egerton, Greg Harris, Mickael Maison) and 3 non-binding votes (Andrew Schofield, Yash Mayya, Kamal Chandraprakash). I'll update the KIP status, meanwhile the PR is still pending: https://github.com/apache/kafka/pull/14093 From: dev@kafka.apache.org At: 08/08/23 08:33:21 UTC-4:00To: dev@kafka.apache.org Subject: Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect Hi, +1 (binding) Thanks for the KIP! On Mon, Aug 7, 2023 at 3:15 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) wrote: > > Hello, > > I still need help from a committer to review/approve this (small) KIP, which adds a new BooleanConverter to the list of converters in Kafka Connect. > > The KIP has a companion PR implementing the feature as well. > > Thanks again! > Sent from Bloomberg Professional for iPhone > > - Original Message - > From: Hector Geraldino > To: dev@kafka.apache.org > At: 08/01/23 11:48:23 UTC-04:00 > > > Hi, > > Still missing one binding vote for this (very small) KIP to pass :) > > From: dev@kafka.apache.org At: 07/28/23 09:37:45 UTC-4:00To: dev@kafka.apache.org > Subject: Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect > > Hi everyone, > > Thanks everyone who has reviewed and voted for this KIP. > > So far it has received 3 non-binding votes (Andrew Schofield, Yash Mayya, Kamal > Chandraprakash) and 2 binding votes (Chris Egerton, Greg Harris)- still shy of > one binding vote to pass. > > Can we get help from a committer to push it through? > > Thank you! > Hector > > Sent from Bloomberg Professional for iPhone > > - Original Message - > From: Greg Harris > To: dev@kafka.apache.org > At: 07/26/23 12:23:20 UTC-04:00 > > > Hey Hector, > > Thanks for the straightforward and clear KIP! > +1 (binding) > > Thanks, > Greg > > On Wed, Jul 26, 2023 at 5:16 AM Chris Egerton wrote: > > > > +1 (binding) > > > > Thanks Hector! > > > > On Wed, Jul 26, 2023 at 3:18 AM Kamal Chandraprakash < > > kamal.chandraprak...@gmail.com> wrote: > > > > > +1 (non-binding). Thanks for the KIP! > > > > > > On Tue, Jul 25, 2023 at 11:12 PM Yash Mayya wrote: > > > > > > > Hi Hector, > > > > > > > > Thanks for the KIP! > > > > > > > > +1 (non-binding) > > > > > > > > Thanks, > > > > Yash > > > > > > > > On Tue, Jul 25, 2023 at 11:01 PM Andrew Schofield < > > > > andrew_schofield_j...@outlook.com> wrote: > > > > > > > > > Thanks for the KIP. As you say, not that controversial. > > > > > > > > > > +1 (non-binding) > > > > > > > > > > Thanks, > > > > > Andrew > > > > > > > > > > > On 25 Jul 2023, at 18:22, Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > > > > hgerald...@bloomberg.net> wrote: > > > > > > > > > > > > Hi everyone, > > > > > > > > > > > > The changes proposed by KIP-959 (Add BooleanConverter to Kafka > > > Connect) > > > > > have a limited scope and shouldn't be controversial. I'm opening a > > > voting > > > > > thread with the hope that it can be included in the next upcoming 3.6 > > > > > release. > > > > > > > > > > > > Here are some links: > > > > > > > > > > > > KIP: > > > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-959%3A+Add+BooleanConverte > r+to+Kafka+Connect > > > > > > JIRA: https://issues.apache.org/jira/browse/KAFKA-15248 > > > > > > Discussion thread: > > > > > https://lists.apache.org/thread/15c2t0kl9bozmzjxmkl5n57kv4l4o1dt > > > > > > Pull Request: https://github.com/apache/kafka/pull/14093 > > > > > > > > > > > > Thanks! > > > > > > > > > > > > > > > > > > > > > >
Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect
Hi, +1 (binding) Thanks for the KIP! On Mon, Aug 7, 2023 at 3:15 PM Hector Geraldino (BLOOMBERG/ 919 3RD A) wrote: > > Hello, > > I still need help from a committer to review/approve this (small) KIP, which > adds a new BooleanConverter to the list of converters in Kafka Connect. > > The KIP has a companion PR implementing the feature as well. > > Thanks again! > Sent from Bloomberg Professional for iPhone > > - Original Message - > From: Hector Geraldino > To: dev@kafka.apache.org > At: 08/01/23 11:48:23 UTC-04:00 > > > Hi, > > Still missing one binding vote for this (very small) KIP to pass :) > > From: dev@kafka.apache.org At: 07/28/23 09:37:45 UTC-4:00To: > dev@kafka.apache.org > Subject: Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect > > Hi everyone, > > Thanks everyone who has reviewed and voted for this KIP. > > So far it has received 3 non-binding votes (Andrew Schofield, Yash Mayya, > Kamal > Chandraprakash) and 2 binding votes (Chris Egerton, Greg Harris)- still shy of > one binding vote to pass. > > Can we get help from a committer to push it through? > > Thank you! > Hector > > Sent from Bloomberg Professional for iPhone > > - Original Message - > From: Greg Harris > To: dev@kafka.apache.org > At: 07/26/23 12:23:20 UTC-04:00 > > > Hey Hector, > > Thanks for the straightforward and clear KIP! > +1 (binding) > > Thanks, > Greg > > On Wed, Jul 26, 2023 at 5:16 AM Chris Egerton wrote: > > > > +1 (binding) > > > > Thanks Hector! > > > > On Wed, Jul 26, 2023 at 3:18 AM Kamal Chandraprakash < > > kamal.chandraprak...@gmail.com> wrote: > > > > > +1 (non-binding). Thanks for the KIP! > > > > > > On Tue, Jul 25, 2023 at 11:12 PM Yash Mayya wrote: > > > > > > > Hi Hector, > > > > > > > > Thanks for the KIP! > > > > > > > > +1 (non-binding) > > > > > > > > Thanks, > > > > Yash > > > > > > > > On Tue, Jul 25, 2023 at 11:01 PM Andrew Schofield < > > > > andrew_schofield_j...@outlook.com> wrote: > > > > > > > > > Thanks for the KIP. As you say, not that controversial. > > > > > > > > > > +1 (non-binding) > > > > > > > > > > Thanks, > > > > > Andrew > > > > > > > > > > > On 25 Jul 2023, at 18:22, Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > > > > hgerald...@bloomberg.net> wrote: > > > > > > > > > > > > Hi everyone, > > > > > > > > > > > > The changes proposed by KIP-959 (Add BooleanConverter to Kafka > > > Connect) > > > > > have a limited scope and shouldn't be controversial. I'm opening a > > > voting > > > > > thread with the hope that it can be included in the next upcoming 3.6 > > > > > release. > > > > > > > > > > > > Here are some links: > > > > > > > > > > > > KIP: > > > > > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-959%3A+Add+BooleanConverte > r+to+Kafka+Connect > > > > > > JIRA: https://issues.apache.org/jira/browse/KAFKA-15248 > > > > > > Discussion thread: > > > > > https://lists.apache.org/thread/15c2t0kl9bozmzjxmkl5n57kv4l4o1dt > > > > > > Pull Request: https://github.com/apache/kafka/pull/14093 > > > > > > > > > > > > Thanks! > > > > > > > > > > > > > > > > > > > > > >
Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect
Hello, I still need help from a committer to review/approve this (small) KIP, which adds a new BooleanConverter to the list of converters in Kafka Connect. The KIP has a companion PR implementing the feature as well. Thanks again! Sent from Bloomberg Professional for iPhone - Original Message - From: Hector Geraldino To: dev@kafka.apache.org At: 08/01/23 11:48:23 UTC-04:00 Hi, Still missing one binding vote for this (very small) KIP to pass :) From: dev@kafka.apache.org At: 07/28/23 09:37:45 UTC-4:00To: dev@kafka.apache.org Subject: Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect Hi everyone, Thanks everyone who has reviewed and voted for this KIP. So far it has received 3 non-binding votes (Andrew Schofield, Yash Mayya, Kamal Chandraprakash) and 2 binding votes (Chris Egerton, Greg Harris)- still shy of one binding vote to pass. Can we get help from a committer to push it through? Thank you! Hector Sent from Bloomberg Professional for iPhone - Original Message - From: Greg Harris To: dev@kafka.apache.org At: 07/26/23 12:23:20 UTC-04:00 Hey Hector, Thanks for the straightforward and clear KIP! +1 (binding) Thanks, Greg On Wed, Jul 26, 2023 at 5:16 AM Chris Egerton wrote: > > +1 (binding) > > Thanks Hector! > > On Wed, Jul 26, 2023 at 3:18 AM Kamal Chandraprakash < > kamal.chandraprak...@gmail.com> wrote: > > > +1 (non-binding). Thanks for the KIP! > > > > On Tue, Jul 25, 2023 at 11:12 PM Yash Mayya wrote: > > > > > Hi Hector, > > > > > > Thanks for the KIP! > > > > > > +1 (non-binding) > > > > > > Thanks, > > > Yash > > > > > > On Tue, Jul 25, 2023 at 11:01 PM Andrew Schofield < > > > andrew_schofield_j...@outlook.com> wrote: > > > > > > > Thanks for the KIP. As you say, not that controversial. > > > > > > > > +1 (non-binding) > > > > > > > > Thanks, > > > > Andrew > > > > > > > > > On 25 Jul 2023, at 18:22, Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > > > hgerald...@bloomberg.net> wrote: > > > > > > > > > > Hi everyone, > > > > > > > > > > The changes proposed by KIP-959 (Add BooleanConverter to Kafka > > Connect) > > > > have a limited scope and shouldn't be controversial. I'm opening a > > voting > > > > thread with the hope that it can be included in the next upcoming 3.6 > > > > release. > > > > > > > > > > Here are some links: > > > > > > > > > > KIP: > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-959%3A+Add+BooleanConverte r+to+Kafka+Connect > > > > > JIRA: https://issues.apache.org/jira/browse/KAFKA-15248 > > > > > Discussion thread: > > > > https://lists.apache.org/thread/15c2t0kl9bozmzjxmkl5n57kv4l4o1dt > > > > > Pull Request: https://github.com/apache/kafka/pull/14093 > > > > > > > > > > Thanks! > > > > > > > > > > > > > > > > >
[jira] [Resolved] (KAFKA-8597) Give access to the Dead Letter Queue APIs to Kafka Connect Developers
[ https://issues.apache.org/jira/browse/KAFKA-8597?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Greg Harris resolved KAFKA-8597. Fix Version/s: 2.6.0 Resolution: Fixed > Give access to the Dead Letter Queue APIs to Kafka Connect Developers > - > > Key: KAFKA-8597 > URL: https://issues.apache.org/jira/browse/KAFKA-8597 > Project: Kafka > Issue Type: Improvement > Components: KafkaConnect >Reporter: Andrea Santurbano >Priority: Major > Labels: needs-kip > Fix For: 2.6.0 > > > Would be cool to have the chance to have access to the DLQ APIs in order to > enable us (developers) to use that. > For instance, if someone uses JSON as message format with no schema and it's > trying to import some data into a table, and the JSON contains a null value > for a NON-NULL table field, so we want to move that event to the DLQ. > Thanks a lot! -- This message was sent by Atlassian Jira (v8.20.10#820010)
Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect
Hi, Still missing one binding vote for this (very small) KIP to pass :) From: dev@kafka.apache.org At: 07/28/23 09:37:45 UTC-4:00To: dev@kafka.apache.org Subject: Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect Hi everyone, Thanks everyone who has reviewed and voted for this KIP. So far it has received 3 non-binding votes (Andrew Schofield, Yash Mayya, Kamal Chandraprakash) and 2 binding votes (Chris Egerton, Greg Harris)- still shy of one binding vote to pass. Can we get help from a committer to push it through? Thank you! Hector Sent from Bloomberg Professional for iPhone - Original Message - From: Greg Harris To: dev@kafka.apache.org At: 07/26/23 12:23:20 UTC-04:00 Hey Hector, Thanks for the straightforward and clear KIP! +1 (binding) Thanks, Greg On Wed, Jul 26, 2023 at 5:16 AM Chris Egerton wrote: > > +1 (binding) > > Thanks Hector! > > On Wed, Jul 26, 2023 at 3:18 AM Kamal Chandraprakash < > kamal.chandraprak...@gmail.com> wrote: > > > +1 (non-binding). Thanks for the KIP! > > > > On Tue, Jul 25, 2023 at 11:12 PM Yash Mayya wrote: > > > > > Hi Hector, > > > > > > Thanks for the KIP! > > > > > > +1 (non-binding) > > > > > > Thanks, > > > Yash > > > > > > On Tue, Jul 25, 2023 at 11:01 PM Andrew Schofield < > > > andrew_schofield_j...@outlook.com> wrote: > > > > > > > Thanks for the KIP. As you say, not that controversial. > > > > > > > > +1 (non-binding) > > > > > > > > Thanks, > > > > Andrew > > > > > > > > > On 25 Jul 2023, at 18:22, Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > > > hgerald...@bloomberg.net> wrote: > > > > > > > > > > Hi everyone, > > > > > > > > > > The changes proposed by KIP-959 (Add BooleanConverter to Kafka > > Connect) > > > > have a limited scope and shouldn't be controversial. I'm opening a > > voting > > > > thread with the hope that it can be included in the next upcoming 3.6 > > > > release. > > > > > > > > > > Here are some links: > > > > > > > > > > KIP: > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-959%3A+Add+BooleanConverte r+to+Kafka+Connect > > > > > JIRA: https://issues.apache.org/jira/browse/KAFKA-15248 > > > > > Discussion thread: > > > > https://lists.apache.org/thread/15c2t0kl9bozmzjxmkl5n57kv4l4o1dt > > > > > Pull Request: https://github.com/apache/kafka/pull/14093 > > > > > > > > > > Thanks! > > > > > > > > > > > > > > > > >
Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect
Hi everyone, Thanks everyone who has reviewed and voted for this KIP. So far it has received 3 non-binding votes (Andrew Schofield, Yash Mayya, Kamal Chandraprakash) and 2 binding votes (Chris Egerton, Greg Harris)- still shy of one binding vote to pass. Can we get help from a committer to push it through? Thank you! Hector Sent from Bloomberg Professional for iPhone - Original Message - From: Greg Harris To: dev@kafka.apache.org At: 07/26/23 12:23:20 UTC-04:00 Hey Hector, Thanks for the straightforward and clear KIP! +1 (binding) Thanks, Greg On Wed, Jul 26, 2023 at 5:16 AM Chris Egerton wrote: > > +1 (binding) > > Thanks Hector! > > On Wed, Jul 26, 2023 at 3:18 AM Kamal Chandraprakash < > kamal.chandraprak...@gmail.com> wrote: > > > +1 (non-binding). Thanks for the KIP! > > > > On Tue, Jul 25, 2023 at 11:12 PM Yash Mayya wrote: > > > > > Hi Hector, > > > > > > Thanks for the KIP! > > > > > > +1 (non-binding) > > > > > > Thanks, > > > Yash > > > > > > On Tue, Jul 25, 2023 at 11:01 PM Andrew Schofield < > > > andrew_schofield_j...@outlook.com> wrote: > > > > > > > Thanks for the KIP. As you say, not that controversial. > > > > > > > > +1 (non-binding) > > > > > > > > Thanks, > > > > Andrew > > > > > > > > > On 25 Jul 2023, at 18:22, Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > > > hgerald...@bloomberg.net> wrote: > > > > > > > > > > Hi everyone, > > > > > > > > > > The changes proposed by KIP-959 (Add BooleanConverter to Kafka > > Connect) > > > > have a limited scope and shouldn't be controversial. I'm opening a > > voting > > > > thread with the hope that it can be included in the next upcoming 3.6 > > > > release. > > > > > > > > > > Here are some links: > > > > > > > > > > KIP: > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-959%3A+Add+BooleanConverter+to+Kafka+Connect > > > > > JIRA: https://issues.apache.org/jira/browse/KAFKA-15248 > > > > > Discussion thread: > > > > https://lists.apache.org/thread/15c2t0kl9bozmzjxmkl5n57kv4l4o1dt > > > > > Pull Request: https://github.com/apache/kafka/pull/14093 > > > > > > > > > > Thanks! > > > > > > > > > > > > > > > > >
Re: [VOTE] KIP-959 Add BooleanConverter to Kafka Connect
Hey Hector, Thanks for the straightforward and clear KIP! +1 (binding) Thanks, Greg On Wed, Jul 26, 2023 at 5:16 AM Chris Egerton wrote: > > +1 (binding) > > Thanks Hector! > > On Wed, Jul 26, 2023 at 3:18 AM Kamal Chandraprakash < > kamal.chandraprak...@gmail.com> wrote: > > > +1 (non-binding). Thanks for the KIP! > > > > On Tue, Jul 25, 2023 at 11:12 PM Yash Mayya wrote: > > > > > Hi Hector, > > > > > > Thanks for the KIP! > > > > > > +1 (non-binding) > > > > > > Thanks, > > > Yash > > > > > > On Tue, Jul 25, 2023 at 11:01 PM Andrew Schofield < > > > andrew_schofield_j...@outlook.com> wrote: > > > > > > > Thanks for the KIP. As you say, not that controversial. > > > > > > > > +1 (non-binding) > > > > > > > > Thanks, > > > > Andrew > > > > > > > > > On 25 Jul 2023, at 18:22, Hector Geraldino (BLOOMBERG/ 919 3RD A) < > > > > hgerald...@bloomberg.net> wrote: > > > > > > > > > > Hi everyone, > > > > > > > > > > The changes proposed by KIP-959 (Add BooleanConverter to Kafka > > Connect) > > > > have a limited scope and shouldn't be controversial. I'm opening a > > voting > > > > thread with the hope that it can be included in the next upcoming 3.6 > > > > release. > > > > > > > > > > Here are some links: > > > > > > > > > > KIP: > > > > > > > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-959%3A+Add+BooleanConverter+to+Kafka+Connect > > > > > JIRA: https://issues.apache.org/jira/browse/KAFKA-15248 > > > > > Discussion thread: > > > > https://lists.apache.org/thread/15c2t0kl9bozmzjxmkl5n57kv4l4o1dt > > > > > Pull Request: https://github.com/apache/kafka/pull/14093 > > > > > > > > > > Thanks! > > > > > > > > > > > > > > > > >