[jira] [Commented] (KAFKA-7228) DeadLetterQueue throws a NullPointerException

2018-08-18 Thread Arjun Satish (JIRA)


[ 
https://issues.apache.org/jira/browse/KAFKA-7228?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16584897#comment-16584897
 ] 

Arjun Satish commented on KAFKA-7228:
-

Installation of hotfix jar:

# Navigate to the lib/ directory of the AK 2.0 installation.
# Replace the "connect-runtime-2.0.0.jar" file with the one provided in this 
ticket:  [^connect-runtime-2.0.0-HOTFIX-7228.jar] .

*Note*: this hotfix only adds the fix for the NPE described in this ticket on 
the 2.0.0 release. Other bugs in connect-runtime 2.0.0 (if any) will not be 
fixed by it.

> DeadLetterQueue throws a NullPointerException
> -
>
> Key: KAFKA-7228
> URL: https://issues.apache.org/jira/browse/KAFKA-7228
> Project: Kafka
>  Issue Type: Task
>  Components: KafkaConnect
>Reporter: Arjun Satish
>Assignee: Arjun Satish
>Priority: Major
> Fix For: 2.0.1, 2.1.0
>
> Attachments: connect-runtime-2.0.0-HOTFIX-7228.jar
>
>
> Using the dead letter queue results in a NPE: 
> {code:java}
> [2018-08-01 07:41:08,907] ERROR WorkerSinkTask{id=s3-sink-yanivr-2-4} Task 
> threw an uncaught and unrecoverable exception 
> (org.apache.kafka.connect.runtime.WorkerTask)
> java.lang.NullPointerException
> at 
> org.apache.kafka.connect.runtime.errors.DeadLetterQueueReporter.report(DeadLetterQueueReporter.java:124)
> at 
> org.apache.kafka.connect.runtime.errors.ProcessingContext.report(ProcessingContext.java:137)
> at 
> org.apache.kafka.connect.runtime.errors.RetryWithToleranceOperator.execute(RetryWithToleranceOperator.java:108)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.convertAndTransformRecord(WorkerSinkTask.java:513)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.convertMessages(WorkerSinkTask.java:490)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.poll(WorkerSinkTask.java:321)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.iteration(WorkerSinkTask.java:225)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.execute(WorkerSinkTask.java:193)
> at org.apache.kafka.connect.runtime.WorkerTask.doRun(WorkerTask.java:175)
> at org.apache.kafka.connect.runtime.WorkerTask.run(WorkerTask.java:219)
> at java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
> at java.util.concurrent.FutureTask.run(FutureTask.java:266)
> at 
> java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1149)
> at 
> java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:624)
> at java.lang.Thread.run(Thread.java:748)
> [2018-08-01 07:41:08,908] ERROR WorkerSinkTask{id=s3-sink-yanivr-2-4} Task is 
> being killed and will not recover until manually restarted 
> (org.apache.kafka.connect.runtime.WorkerTask)
> {code}
> DLQ reporter does not get a {{errorHandlingMetrics}} object when initialized 
> through the WorkerSinkTask.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Commented] (KAFKA-7228) DeadLetterQueue throws a NullPointerException

2018-08-02 Thread ASF GitHub Bot (JIRA)


[ 
https://issues.apache.org/jira/browse/KAFKA-7228?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16567511#comment-16567511
 ] 

ASF GitHub Bot commented on KAFKA-7228:
---

ewencp closed pull request #5440: KAFKA-7228: Set errorHandlingMetrics for dead 
letter queue
URL: https://github.com/apache/kafka/pull/5440
 
 
   

This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:

As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):

diff --git 
a/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java 
b/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java
index 1f62103213c..65d85caf90e 100644
--- a/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java
+++ b/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/Worker.java
@@ -524,14 +524,13 @@ ErrorHandlingMetrics errorHandlingMetrics(ConnectorTaskId 
id) {
 private List sinkTaskReporters(ConnectorTaskId id, 
SinkConnectorConfig connConfig,
   ErrorHandlingMetrics 
errorHandlingMetrics) {
 ArrayList reporters = new ArrayList<>();
-LogReporter logReporter = new LogReporter(id, connConfig);
-logReporter.metrics(errorHandlingMetrics);
+LogReporter logReporter = new LogReporter(id, connConfig, 
errorHandlingMetrics);
 reporters.add(logReporter);
 
 // check if topic for dead letter queue exists
 String topic = connConfig.dlqTopicName();
 if (topic != null && !topic.isEmpty()) {
-DeadLetterQueueReporter reporter = 
DeadLetterQueueReporter.createAndSetup(config, id, connConfig, producerProps);
+DeadLetterQueueReporter reporter = 
DeadLetterQueueReporter.createAndSetup(config, id, connConfig, producerProps, 
errorHandlingMetrics);
 reporters.add(reporter);
 }
 
@@ -541,8 +540,7 @@ ErrorHandlingMetrics errorHandlingMetrics(ConnectorTaskId 
id) {
 private List sourceTaskReporters(ConnectorTaskId id, 
ConnectorConfig connConfig,
   ErrorHandlingMetrics 
errorHandlingMetrics) {
 List reporters = new ArrayList<>();
-LogReporter logReporter = new LogReporter(id, connConfig);
-logReporter.metrics(errorHandlingMetrics);
+LogReporter logReporter = new LogReporter(id, connConfig, 
errorHandlingMetrics);
 reporters.add(logReporter);
 
 return reporters;
diff --git 
a/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/errors/DeadLetterQueueReporter.java
 
b/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/errors/DeadLetterQueueReporter.java
index d36ec22ec88..c059dcff793 100644
--- 
a/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/errors/DeadLetterQueueReporter.java
+++ 
b/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/errors/DeadLetterQueueReporter.java
@@ -36,6 +36,7 @@
 import java.io.PrintStream;
 import java.nio.charset.StandardCharsets;
 import java.util.Map;
+import java.util.Objects;
 import java.util.concurrent.ExecutionException;
 
 import static java.util.Collections.singleton;
@@ -66,13 +67,14 @@
 
 private final SinkConnectorConfig connConfig;
 private final ConnectorTaskId connectorTaskId;
+private final ErrorHandlingMetrics errorHandlingMetrics;
 
 private KafkaProducer kafkaProducer;
-private ErrorHandlingMetrics errorHandlingMetrics;
 
 public static DeadLetterQueueReporter createAndSetup(WorkerConfig 
workerConfig,
  ConnectorTaskId id,
- SinkConnectorConfig 
sinkConfig, Map producerProps) {
+ SinkConnectorConfig 
sinkConfig, Map producerProps,
+ ErrorHandlingMetrics 
errorHandlingMetrics) {
 String topic = sinkConfig.dlqTopicName();
 
 try (AdminClient admin = AdminClient.create(workerConfig.originals())) 
{
@@ -90,7 +92,7 @@ public static DeadLetterQueueReporter 
createAndSetup(WorkerConfig workerConfig,
 }
 
 KafkaProducer dlqProducer = new 
KafkaProducer<>(producerProps);
-return new DeadLetterQueueReporter(dlqProducer, sinkConfig, id);
+return new DeadLetterQueueReporter(dlqProducer, sinkConfig, id, 
errorHandlingMetrics);
 }
 
 /**
@@ -99,14 +101,16 @@ public static DeadLetterQueueReporter 
createAndSetup(WorkerConfig workerConfig,
  * @param kafkaProducer a Kafka Producer to produce the original consumed 
records.
  */
 // Visible for testing
-DeadLetterQueueReporter(KafkaProducer kafkaProducer, 
SinkConnectorConfig 

[jira] [Commented] (KAFKA-7228) DeadLetterQueue throws a NullPointerException

2018-08-01 Thread ASF GitHub Bot (JIRA)


[ 
https://issues.apache.org/jira/browse/KAFKA-7228?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16565540#comment-16565540
 ] 

ASF GitHub Bot commented on KAFKA-7228:
---

wicknicks opened a new pull request #5440: KAFKA-7228: Set errorHandlingMetrics 
for dead letter queue
URL: https://github.com/apache/kafka/pull/5440
 
 
   DLQ reporter does not get a errorHandlingMetrics object when created by the 
worker.
   
   Signed-off-by: Arjun Satish 
   
   *More detailed description of your change,
   if necessary. The PR title and PR message become
   the squashed commit message, so use a separate
   comment to ping reviewers.*
   
   *Summary of testing strategy (including rationale)
   for the feature or bug fix. Unit and/or integration
   tests are expected for any behaviour change and
   system tests should be considered for larger changes.*
   
   ### Committer Checklist (excluded from commit message)
   - [ ] Verify design and implementation 
   - [ ] Verify test coverage and CI build status
   - [ ] Verify documentation (including upgrade notes)
   


This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


> DeadLetterQueue throws a NullPointerException
> -
>
> Key: KAFKA-7228
> URL: https://issues.apache.org/jira/browse/KAFKA-7228
> Project: Kafka
>  Issue Type: Task
>  Components: KafkaConnect
>Reporter: Arjun Satish
>Assignee: Arjun Satish
>Priority: Major
>
> Using the dead letter queue results in a NPE: 
> {code:java}
> [2018-08-01 07:41:08,907] ERROR WorkerSinkTask{id=s3-sink-yanivr-2-4} Task 
> threw an uncaught and unrecoverable exception 
> (org.apache.kafka.connect.runtime.WorkerTask)
> java.lang.NullPointerException
> at 
> org.apache.kafka.connect.runtime.errors.DeadLetterQueueReporter.report(DeadLetterQueueReporter.java:124)
> at 
> org.apache.kafka.connect.runtime.errors.ProcessingContext.report(ProcessingContext.java:137)
> at 
> org.apache.kafka.connect.runtime.errors.RetryWithToleranceOperator.execute(RetryWithToleranceOperator.java:108)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.convertAndTransformRecord(WorkerSinkTask.java:513)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.convertMessages(WorkerSinkTask.java:490)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.poll(WorkerSinkTask.java:321)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.iteration(WorkerSinkTask.java:225)
> at 
> org.apache.kafka.connect.runtime.WorkerSinkTask.execute(WorkerSinkTask.java:193)
> at org.apache.kafka.connect.runtime.WorkerTask.doRun(WorkerTask.java:175)
> at org.apache.kafka.connect.runtime.WorkerTask.run(WorkerTask.java:219)
> at java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
> at java.util.concurrent.FutureTask.run(FutureTask.java:266)
> at 
> java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1149)
> at 
> java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:624)
> at java.lang.Thread.run(Thread.java:748)
> [2018-08-01 07:41:08,908] ERROR WorkerSinkTask{id=s3-sink-yanivr-2-4} Task is 
> being killed and will not recover until manually restarted 
> (org.apache.kafka.connect.runtime.WorkerTask)
> {code}
> DLQ reporter does not get a {{errorHandlingMetrics}} object when initialized 
> through the WorkerSinkTask.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)