Mark, I’ve verified the fix in 0.8-SNAPSHOT. I’ve resolved NIFI-2680.
Thanks, Chris McDermott Remote Business Analytics STaTS/StoreFront Remote HPE Storage Hewlett Packard Enterprise Mobile: +1 978-697-5315 On 9/9/16, 12:17 PM, "Mark Payne" <marka...@hotmail.com> wrote: Chris, Ah, sorry about that - I saw "KafkaPublisher" and thought "PublishKafka." Yes, please do test with a newer version, as I believe the bug was fixed in 0.8.0 and 1.0.0. Thanks! -Mark > On Sep 9, 2016, at 11:32 AM, McDermott, Chris Kevin (MSDU - STaTS/StorefrontRemote) <chris.mcderm...@hpe.com> wrote: > > Hi Mark, > > Thanks for the quick reply. > > I am definitely using PutKafka. If you really squint at the stack trace you can see > >> at org.apache.nifi.processors.kafka.KafkaPublisher.publish(KafkaPublisher.java:140) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.processors.kafka.PutKafka$1.process(PutKafka.java:315) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] > > I made a bad assumption about NIFI-2680 affecting 0.8.0. > > I will retest with 0.8.0 and update NIFI-2680 accordingly. > > Thanks, > > Chris McDermott > > Remote Business Analytics > STaTS/StoreFront Remote > HPE Storage > Hewlett Packard Enterprise > Mobile: +1 978-697-5315 > > > > > On 9/9/16, 9:46 AM, "Mark Payne" <marka...@hotmail.com> wrote: > > Chris, > > I was looking into this, but have a few questions for you. > > In NIFI-2680, it indicates that the problem was with PutKafka. This email seems to indicate that you're seeing a problem with PublishKafka. > So just want to ensure that we're on the same page - are you seeing issues in both PutKafka and PublishKafka? > > I have been testing with PutKafka and putting a 5 MB FlowFile does indeed fail with an error message saying that the FlowFile is too large. > It appears that this was fixed in NIFI-2445, which has gone into 0.8.0 and 1.0.0. However, NIFI-2680 indicates that the problem affects 0.7.0, > 0.8.0, and 1.0.0. > > From your stack trace here it appears that you're using Kafka Processors from an 0.7.x release. Have you verified that this issue still exists > with a newer version of NiFi? > > Thanks! > -Mark > > > >> On Sep 8, 2016, at 8:00 PM, McDermott, Chris Kevin (MSDU - STaTS/StorefrontRemote) <chris.mcderm...@hpe.com> wrote: >> >> Hello dev folks. >> >> Would it be possible for someone to take a look at NIFI-2680? https://issues.apache.org/jira/browse/NIFI-2680?jql=project%20%3D%20NIFI%20AND%20reporter%20%3D%20currentUser() <https://issues.apache.org/jira/browse/NIFI-2680?jql=project%20%3D%20NIFI%20AND%20reporter%20%3D%20currentUser()> >> I believe I have a similar situation where writing to Kafka fails when the Kafka broker is restarted. The flowfile never seems to make it to Kafka but it transferred to the PutKafka success output anyway. i.e. this is a data-loss bug. >> >> Here is the traceback from the failure. >> >> 2016-09-08 18:41:44,507 ERROR [Timer-Driven Process Thread-8] o.a.nifi.processors.kafka.KafkaPublisher Failed while waiting for acks from Kafka >> java.util.concurrent.ExecutionException: org.apache.kafka.common.errors.NotLeaderForPartitionException: This server is not the leader for that topic-partition. >> at org.apache.kafka.clients.producer.internals.FutureRecordMetadata.valueOrError(FutureRecordMetadata.java:56) ~[kafka-clients-0.8.2.2.jar:na] >> at org.apache.kafka.clients.producer.internals.FutureRecordMetadata.get(FutureRecordMetadata.java:51) ~[kafka-clients-0.8.2.2.jar:na] >> at org.apache.kafka.clients.producer.internals.FutureRecordMetadata.get(FutureRecordMetadata.java:25) ~[kafka-clients-0.8.2.2.jar:na] >> at org.apache.nifi.processors.kafka.KafkaPublisher.processAcks(KafkaPublisher.java:197) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.processors.kafka.KafkaPublisher.publish(KafkaPublisher.java:140) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.processors.kafka.PutKafka$1.process(PutKafka.java:315) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.controller.repository.StandardProcessSession.read(StandardProcessSession.java:1851) [nifi-framework-core-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.controller.repository.StandardProcessSession.read(StandardProcessSession.java:1822) [nifi-framework-core-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.processors.kafka.PutKafka.doRendezvousWithKafka(PutKafka.java:311) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.processors.kafka.PutKafka.rendezvousWithKafka(PutKafka.java:287) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.processors.kafka.AbstractKafkaProcessor.onTrigger(AbstractKafkaProcessor.java:76) [nifi-kafka-processors-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.controller.StandardProcessorNode.onTrigger(StandardProcessorNode.java:1054) [nifi-framework-core-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.controller.tasks.ContinuallyRunProcessorTask.call(ContinuallyRunProcessorTask.java:136) [nifi-framework-core-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.controller.tasks.ContinuallyRunProcessorTask.call(ContinuallyRunProcessorTask.java:47) [nifi-framework-core-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at org.apache.nifi.controller.scheduling.TimerDrivenSchedulingAgent$1.run(TimerDrivenSchedulingAgent.java:127) [nifi-framework-core-oculus-0.7.x-SNAPSHOT.jar:oculus-0.7.x-SNAPSHOT] >> at java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511) [na:1.8.0_45] >> at java.util.concurrent.FutureTask.runAndReset(FutureTask.java:308) [na:1.8.0_45] >> at java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.access$301(ScheduledThreadPoolExecutor.java:180) [na:1.8.0_45] >> at java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.run(ScheduledThreadPoolExecutor.java:294) [na:1.8.0_45] >> at java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1142) [na:1.8.0_45] >> at java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:617) [na:1.8.0_45] >> at java.lang.Thread.run(Thread.java:745) [na:1.8.0_45] >> Caused by: org.apache.kafka.common.errors.NotLeaderForPartitionException: This server is not the leader for that topic-partition. >> >> Thanks, >> >> Chris McDermott >> >> Remote Business Analytics >> STaTS/StoreFront Remote >> HPE Storage >> Hewlett Packard Enterprise >> Mobile: +1 978-697-5315 >> >> > > > >