Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-21 Thread Bill Bejeck
+1 for the approach outlined above by Eno. On Wed, Jun 21, 2017 at 11:28 AM, Damian Guy wrote: > Thanks Eno. > > Yes i agree. We could apply this same approach to most of the operations > where we have multiple overloads, i.e., we have a single method for each > operation that takes the required

Re: Weird broker lock-up causing an almost global downtime

2017-06-27 Thread Bill Bejeck
On 2017-06-27 14:46 (-0400), Vincent Rischmann wrote: > Hello. so I had a weird problem this afternoon. I was deploying a > streams application and wanted to delete already existing internal > states data so I ran kafka-streams-application-reset.sh to do it, as > recommended. it wasn't the firs

Re: Weird broker lock-up causing an almost global downtime

2017-06-27 Thread Bill Bejeck
Hi Vincent, Thanks for reporting this issue. Could you give us some more details (number topics, partitions per topic and the structure of your Kafka Streams application) so we attempt to reproduce and diagnose the issue? Thanks! Bill On 2017-06-27 14:46 (-0400), Vincent Rischmann wrote: >

Re: Weird broker lock-up causing an almost global downtime

2017-06-27 Thread Bill Bejeck
Hi Vincent, Thanks for reporting this. Could you give some details on your setup (topics, partitions and the structure of your streams application) so I can attempt to reproduce the issue? Thanks! On 2017-06-27 14:46 (-0400), Vincent Rischmann wrote: > Hello. so I had a weird problem this a

Re: Weird broker lock-up causing an almost global downtime

2017-06-27 Thread Bill Bejeck
> - 1 day window, 15 days retention > That's basically it for the structure. > The input topic has 64 partitions. > > Tomorrow I can get some logs from Kafka/Zookeeper if that would help. > > On Tue, Jun 27, 2017, at 11:41 PM, Bill Bejeck wrote: > > Hi Vincent, > &g

Re: kafka-streams app(s) stopped consuming new events

2017-06-28 Thread Bill Bejeck
Hi Dimitry, I'm happy to help, but I could use more information. Can you share the streams logs and broker logs? Have you confirmed messages are still being delivered to topics (via console consumer)? Thanks, Bill On Wed, Jun 28, 2017 at 8:24 AM, Dmitriy Vsekhvalnov wrote: > Hi all, > > look

Re: kafka-streams app(s) stopped consuming new events

2017-06-28 Thread Bill Bejeck
StreamTask 0_0] > > [2017-06-27 03:30:50,554] [] [INFO ] [StreamThread-1] > [org.apache.kafka.streams.processor.internals.StreamThread] [stream-thread > [StreamThread-1] Committing task StreamTask 2_0] > > Something specific to search for? > > 2. Yes, there are more messag

Re: kafka-streams app(s) stopped consuming new events

2017-06-28 Thread Bill Bejeck
st.github.com/dvsekhvalnov/8e870f7347394e8d004c282880ef38 > 5a#file-kafka-broker-1-2-3-log > > All broker logs are same, so single gist. > > There are also state-change.log files, do you want to take a look at those > as well? > > On Wed, Jun 28, 2017 at 4:31 PM, Bill Bejeck wrote: > > > Hi D

Re: Weird broker lock-up causing an almost global downtime

2017-06-28 Thread Bill Bejeck
I tried to delete: > https://pastebin.com/raw/bz9MYRsJ > Then just one minute later it deleted the topics > https://pastebin.com/raw/EQ2Kuwj6 > > And after that it operated normally. > > Sorry if it's a bit random and unorganized, I hope it helps anyway. I > can search for so

Re: Weird broker lock-up causing an almost global downtime

2017-06-29 Thread Bill Bejeck
Hi Vincent, What version of Kafka/Kafka Streams are you running, more specifically when this error occurred? Thanks, Bill On Wed, Jun 28, 2017 at 12:24 PM, Bill Bejeck wrote: > Thanks for the info Vincent. > > -Bill > > On Wed, Jun 28, 2017 at 12:19 PM, Vincent Rischmann &g

Re: KIP-162: Enable topic deletion by default

2017-07-19 Thread Bill Bejeck
+1 On Wed, Jul 19, 2017 at 10:51 AM, Damian Guy wrote: > +1 > > On Wed, 31 May 2017 at 13:36 Jim Jagielski wrote: > > > +1 > > > On May 27, 2017, at 9:27 PM, Vahid S Hashemian < > > vahidhashem...@us.ibm.com> wrote: > > > > > > Sure, that sounds good. > > > > > > I suggested that to keep comman

Re: Kafka Streams: why aren't offsets being committed?

2017-07-20 Thread Bill Bejeck
Hi Dmitry, When you say "even if the application has not had data for a long time" do you have a rough idea of how long? What is the value of your "auto.offset.reset" configuration? Thanks, Bill On Thu, Jul 20, 2017 at 6:03 PM, Dmitry Minkovsky wrote: > My Streams application is configured

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-08-02 Thread Bill Bejeck
Hi Jan, Thanks for the effort in putting your thoughts down on paper. Comparing what I see from your proposal and what is presented in KIP-182, one of the main differences is the exclusion of an`Materialized` instance in the `KTable` methods. Can you go into more detail why this is so and the s

Re: Classloading Error with Kotlin and Streams

2017-09-08 Thread Bill Bejeck
Hi Avi, I'm not super familiar with Kotiln, but I'm taking a look. Thanks, Bill On Fri, Sep 8, 2017 at 4:28 PM, Avi Flax wrote: > Hi all, I’m trying to experiment with Kotlin and Streams, and I’m > encountering an error: > > $ kotlinc -cp > kafka-streams-0.11.0.0-cp1.jar:kafka-clients-0.11.0.

Re: Classloading Error with Kotlin and Streams

2017-09-11 Thread Bill Bejeck
I think you'll need to put a JIRA ticket in, sorry this isn't very helpful at this point. Thanks, Bill On Mon, Sep 11, 2017 at 10:53 AM, Avi Flax wrote: > > > On Sep 8, 2017, at 17:57, Bill Bejeck wrote: > > > > I'm not super familiar with Kotiln, but I&#x

Re: Kafka Streams application Unable to Horizontally scale and the application on other instances refusing to start.

2017-09-15 Thread Bill Bejeck
Hi, Could you set MAX_POLL_RECORDS to something lower like 500 or 1000 and try again? Thanks, Bill On Fri, Sep 15, 2017 at 3:40 PM, dev loper wrote: > Hi Damian, > > I have repeated my tests with slight configuration change. The current > logs captured for "StreamThread" keyword has more rele

Re: Improving Kafka State Store performance

2017-09-16 Thread Bill Bejeck
Hi, It's hard to say exactly without a little more information. On a side note, I don't see where you are closing the KeyValueIterator in the code above. Not closing a KeyValueIterator on a Permanent State Store can cause a resource leak over time, so I'd add `iter.close()` right before your `log

Re: Improving Kafka State Store performance

2017-09-18 Thread Bill Bejeck
stance Type : AWS T2 Large > Machine Configuration : 2 VCPU;s, 8gb Ram, Storage : EBS > > Kafka Streams Instance : 3 Kafka Streams Application Instances (Current > CPU Usage 8%- 24%) > > Instance Type : AWS M4 Large > Machine Configuration : 2 VCPU;s, 8gb Ram, Storage

Re: Kafka Streams application Unable to Horizontally scale and the application on other instances refusing to start.

2017-09-18 Thread Bill Bejeck
Hi, I just sent you a follow-up message on the other thread we have going regarding state store performance. I guess we can consider this thread closed and we'll continue working on the State Store thread. Thanks! Bill On Mon, Sep 18, 2017 at 7:27 AM, dev loper wrote: > Hi Ted, Damian, Bill

Re: Improving Kafka State Store performance

2017-09-18 Thread Bill Bejeck
nt using rocksdb > 5.7.3 which is latest. > > Cheers > > On Mon, Sep 18, 2017 at 5:00 AM, Bill Bejeck wrote: > > > I'm following up from your other thread as well here. Thanks for the > info > > above, that is helpful. > > > > I think the AWS insta

Re: kaka-streams 0.11.0.1 rocksdb bug?

2017-09-22 Thread Bill Bejeck
Hi Ara, Can you provide any log files? Thanks, Bill On Fri, Sep 22, 2017 at 4:47 PM, Ara Ebrahimi wrote: > Hi, > > We just upgraded to kaka-streams 0.11.0.1 and noticed that in the cluster > deployment reduce() never gets called. Funny thing is it does gets called > in the unit tests. And no,

Re: Kafka Streams - Custom processor "init" method called before state store has data restored into it

2017-11-09 Thread Bill Bejeck
Hi Rainer, Thanks for reporting this issue. Do you have any log data you can share? In the meantime, I'll look into the issue. Thanks, Bill On Thu, Nov 9, 2017 at 1:23 PM, Rainer Guessner wrote: > I have a custom processor that implements AbstractProcessor and a custom > store that implement

Re: Kafka Streams - Custom processor "init" method called before state store has data restored into it

2017-11-12 Thread Bill Bejeck
ur stuff here > PROCESSOR INIT > 14:21:05,637 INFO ... our stuff here > RESTORE CALLED > ... > RESTORE CALLED > 14:21:05,680 INFO [StreamThread] stream-thread [t10_nr3_metadatarecovery- > 17935bc8-6ec2-4fcc-b62f-15a63c9a051c-StreamThread-1] State transition > from PARTITIO

Re: Kafka Streams - Custom processor "init" method called before state store has data restored into it

2017-11-13 Thread Bill Bejeck
.init > 19:04:36,690 INFO [KStreamsTester$MyStateStore] .onRestoreStart > 19:04:36,706 INFO [KStreamsTester$MyStateStore] .restore > 19:04:36,706 INFO [KStreamsTester$MyStateStore] .restore > 19:04:36,706 INFO [KStreamsTester$MyStateStore] .restore > 19:04:36,706 INFO [KStreamsT

Re: Kafka streams - regarding WordCountInteractiveQueriesExample

2017-11-29 Thread Bill Bejeck
Giridhar, Thanks for reporting this, I'll take a look. On Wed, Nov 29, 2017 at 5:37 AM, Giridhar Addepalli wrote: > Hi, > > I am newbie to Kafka streams. > > Tried below example : > > https://github.com/confluentinc/kafka-streams- > examples/blob/4.0.x/src/main/java/io/confluent/examples/ > str

Re: Kafka streams - regarding WordCountInteractiveQueriesExample

2017-12-05 Thread Bill Bejeck
Giridhar, I can confirm, we'll get a patch for this soon. Thanks for reporting. -Bill On Wed, Nov 29, 2017 at 8:21 AM, Bill Bejeck wrote: > Giridhar, > > Thanks for reporting this, I'll take a look. > > On Wed, Nov 29, 2017 at 5:37 AM, Giridhar Addepalli < >

Re: caching permantly disabled in Kafka Streams

2017-12-13 Thread Bill Bejeck
H Artur, Do you have any log files you can share for this issue? Thanks, Bill On Wed, Dec 13, 2017 at 8:15 AM, Artur Mrozowski wrote: > Actually I can see all other properties being set, except for > CACHE_MAX_BYTES_BUFFERING_CONFIG that is null. > I use 0.11.0.2 Kafka Streams. > Has anyone en

Re: deduplication strategy for Kafka Streams DSL

2017-12-13 Thread Bill Bejeck
Hi Artur, The most direct way for deduplication (I'm using the term deduplication to mean records with the same key, but not necessarily the same value, where later records are considered) is to set the CACHE_MAX_BYTES_BUFFERING_CONFIG setting to a value greater than zero. Your other option is t

Re: caching permantly disabled in Kafka Streams

2017-12-13 Thread Bill Bejeck
; Artur > > On Wed, Dec 13, 2017 at 3:44 PM, Bill Bejeck wrote: > > > H Artur, > > > > Do you have any log files you can share for this issue? > > > > Thanks, > > Bill > > > > On Wed, Dec 13, 2017 at 8:15 AM, Artur Mrozowski

Re: Seeking advice on Kafka Streams and Kafka Connect

2017-12-21 Thread Bill Bejeck
Hi Mads, Great question and yes your use case here is an excellent fit for Kafka Streams and Kafka Connect. For step 2 you could use a KStram#flatMap operation to split it up into multiple rows. Regarding a Cassandra connector, there is an existing one: 1. For some background try https://

Re: Consumer group describe issue

2017-12-27 Thread Bill Bejeck
Sahil, Thanks for the patch. In order to have your patch reviewed, please look over the contribution guidelines outlined here https://kafka.apache.org/contributing Thanks, Bill On Tue, Dec 26, 2017 at 12:45 AM, sahil aggarwal wrote: > Attached the patch. If someone can review it will be very

Re: [ANNOUNCE] New committer: Matthias J. Sax

2018-01-12 Thread Bill Bejeck
Congratulations Matthias! Very well deserved On Fri, Jan 12, 2018 at 6:11 PM Vahid S Hashemian wrote: > Congrats Matthias! Well deserved. > > --Vahid > > > > From: Ted Yu > To: d...@kafka.apache.org > Cc: users@kafka.apache.org > Date: 01/12/2018 03:00 PM > Subject:Re: [ANNO

Re: [DISCUSS] KIP-247: Add public test utils for Kafka Streams

2018-01-16 Thread Bill Bejeck
Thanks for the KIP! One meta question: Will users that are currently using the existing testing code with the "classifier:test" approach: 1) have access to the new testing utilities without updating the gradle.build file 2) can they continue to use the current testing code with the cl

Re: [VOTE] KIP-247: Add public test utils for Kafka Streams

2018-01-18 Thread Bill Bejeck
Thanks for the KIP. +1 -Bill On Wed, Jan 17, 2018 at 9:09 PM, Matthias J. Sax wrote: > Hi, > > I would like to start the vote for KIP-247: > https://cwiki.apache.org/confluence/display/KAFKA/KIP- > 247%3A+Add+public+test+utils+for+Kafka+Streams > > > -Matthias > >

Re: Kafka Streams balancing of tasks across nodes

2018-02-07 Thread Bill Bejeck
Russell, Can you share any log files? Thanks, Bill On Wed, Feb 7, 2018 at 5:45 PM, Russell Teabeault < rteabea...@twitter.com.invalid> wrote: > Hi Matthias, > > Thanks for the prompt reply. We have built the kafka-streams jar from the > 1.1 branch and deployed our instances. We are only able

Re: Kafka Streams balancing of tasks across nodes

2018-02-07 Thread Bill Bejeck
2018 at 6:16 PM, Russell Teabeault < rteabea...@twitter.com.invalid> wrote: > Bill, > > I may be able to. > > - What logging level? > - Do you need logs from all the instances? > - Where should I send them? > > -russ > > On Wed, Feb 7, 2018 at 4:12 PM, Bill B

Re: Doubts in KStreams

2018-02-21 Thread Bill Bejeck
Hi Pravin, 1. Fault tolerance means that state stores are backed by topics, changelogs, storing the contents of the state store. For example, in a worst case scenario, your machine crashed destroying all your local state, on starting your Kafka Streams application back up the state stores would

Re: Consumer group describe issue

2018-02-22 Thread Bill Bejeck
Can someone add Sahil to the contributor list? Thanks, Bill On Thu, Feb 22, 2018 at 3:39 AM, sahil aggarwal wrote: > Bill, > > I have created the jira: https://issues.apache.org/jira/browse/KAFKA-6581 > but i am not able to assign it to myself. > > Can you please help? > > > Thanks, > Sahil > >

Re: Kafka Streams : TimeoutException Expiring Records

2018-02-22 Thread Bill Bejeck
Hi Tony, Looks like you have a known issue that KIP-91( https://cwiki.apache.org/confluence/display/KAFKA/KIP-91+Provide+Intuitive+User+Timeouts+in+The+Producer) will address. In the meantime, as a workaround, you could try setting REQUEST_TIMEOUT_MS_CONFIG to a large value (Integer.MAX_VALUE ?)

Re: ProcessorTopology

2018-02-23 Thread Bill Bejeck
Hi Pravin, Assume you have two topologies, A and B. When building topology "A" you'll have a sink node writing to "topic-A-results", and when building topology "B" you'll specify "topic-A-results" as its source. You'll need to create these topics ahead of time. HTH Bill On Fri, Feb 23, 2018

Re: question on doing deduplication with KafkaStreams

2018-02-27 Thread Bill Bejeck
Hi Nicu, To eliminate old records you'll want to add an `until(final long durationMs)` method to your existing code like so: stream.groupByKey(Serdes.String(), Serdes.String()) .windowedBy(SessionWindows.with(TimeUnit.HOURS.toMillis(1)). *until(TimeUnit.HOURS.toMillis(1))*)

Re: [DISCUSS] KIP-267: Add Processor Unit Test Support to Kafka Streams Test Utils

2018-03-09 Thread Bill Bejeck
John, Sorry for the delayed response. Thanks for the KIP, I'm +1 on it, and I don't have any further comments on the KIP itself aside from the comments that others have raised. Regarding the existing MockProcessorContext and its removal in favor of the one added from this KIP, I'm actually in fa

Re: Kafka Streams 1.1.0 - Significant Performance Drop

2018-04-27 Thread Bill Bejeck
Hi Tony, I'll try to address your questions below: 1. While it's not technically "wrong" to commit for each record, you need to keep in mind that calling commit has resource implications for Kafka Streams. Here's a link to our FAQ describing what happens during a commit https://c

[DISCUSS] KIP-295: Add Streams Configuration Allowing for Optional Topology Optimization

2018-05-07 Thread Bill Bejeck
All, I'd like to start a discussion about adding a configuration parameter allowing for the forthcoming topology optimization to be optional via configuration. The KIP can be found here: https://cwiki.apache.org/confluence/display/KAFKA/KIP-295%3A+Add+Streams+Configuration+Allowing+for+Optional+T

[VOTE] KIP-295: Add Streams Configuration Allowing for Optional Topology Optimization

2018-05-15 Thread Bill Bejeck
Hi all, I'd like to start a vote on KIP-295: Add Streams Configuration Allowing for Optional Topology Optimization. KIP wiki page: https://cwiki.apache.org/confluence/display/KAFKA/KIP-295%3A+Add+Streams+Configuration+Allowing+for+Optional+Topology+Optimization Discussion thread: https://www.mai

Re: Forcing un-assignment of partitions for Kafka application

2018-05-23 Thread Bill Bejeck
Hi Edmondo, While technically you can provide a custom KafkaClientProvider and overwrite the configs for the consumer, I don't recommend you do so. Partition and task assignment is much more involved in Kafka Streams than doing so for plain consumers. Why do you want to force partition assignment

[DISCUSS] KIP-312: Add Overloaded StreamsBuilder Build Method to Accept java.util.Properties

2018-06-08 Thread Bill Bejeck
All, I'd like to start the discussion for adding an overloaded method to StreamsBuilder taking a java.util.Properties instance. The KIP is located here : https://cwiki.apache.org/confluence/display/KAFKA/KIP-312%3A+Add+Overloaded+StreamsBuilder+Build+Method+to+Accept+java.util.Properties I look

[VOTE] KIP-312: Add Overloaded StreamsBuilder Build Method to Accept java.util.Properties

2018-06-25 Thread Bill Bejeck
All, I'd like to start a vote for this KIP now. Thanks, Bill

Re: UNKNOWN_PRODUCER_ID error when running Streams WordCount demo with processing.guarantee set to EXACTLY_ONCE

2018-07-19 Thread Bill Bejeck
Hi Thanks for reporting this. Just off the top of my head, I'm thinking it may have to do with using a console producer, but I'll have to take a deeper look. Thanks, Bill On Thu, Jul 19, 2018 at 9:59 AM lambdaliu(刘少波) wrote: > Hi, > > I test the Kafka Streams WordCount demo follow the steps d

Re: Use Kafka Streams for windowing data and processing each window at once

2018-07-19 Thread Bill Bejeck
Hi Bruno, What you are asking is a common request. There is a KIP in the works, https://cwiki.apache.org/confluence/display/KAFKA/KIP-328%3A+Ability+to+suppress+updates+for+KTables, that should suit the requirements you've outlined. In the meantime, I'll see if I can come up with an alternative

Re: UNKNOWN_PRODUCER_ID error when running Streams WordCount demo with processing.guarantee set to EXACTLY_ONCE

2018-07-20 Thread Bill Bejeck
icky, since multiple > producers may be sending to the same inner topic, so just leaving the last > N messages to not be purged may not be safe still). > > > Bill, could we create a JIRA for this? > > > Guozhang > > > On Thu, Jul 19, 2018 at 7:55 AM, Bill Bejeck w

Re: [DISCUSS] KIP-369 Alternative Partitioner to Support "Always Round-Robin" Selection

2018-08-30 Thread Bill Bejeck
Hi, NOTE: I sent this earlier, but that message just went to the dev list. I'm including both users and dev now. Thanks for the KIP. Have you considered using the overloaded ProducerRecord constructor where you can specify the partition? I mention this as an option as I encountered the same i

Re: Kafka streams: topic partitions->consumer 1:1 mapping not happening

2018-08-31 Thread Bill Bejeck
Hi, What version of Kafka Streams are you using? Can you share some log files preferably at the DEBUG level? Thanks, Bill On Thu, Aug 30, 2018 at 9:58 PM kaustubh khasnis wrote: > Hi, > I have written a streams application to talk to topic on cluster of 5 > brokers with 10 partitions. I h

Re: Kafka streams: topic partitions->consumer 1:1 mapping not happening

2018-08-31 Thread Bill Bejeck
; > Thanks > Kaustubh > > On Fri, Aug 31, 2018 at 7:12 PM Bill Bejeck wrote: > > > Hi, > > > > What version of Kafka Streams are you using? Can you share some log > files > > preferably at the DEBUG level? > > > > Thanks, > > Bill >

[DISCUSS] KIP-372: Naming Joins and Grouping

2018-09-12 Thread Bill Bejeck
All I'd like to start a discussion on KIP-372 for the naming of joins and grouping operations in Kafka Streams. The KIP page can be found here: https://cwiki.apache.org/confluence/display/KAFKA/KIP-372%3A+Naming+Joins+and+Grouping I look forward to feedback and comments. Thanks, Bill

Re: Kafka stream issue : Deleting obsolete state directory

2018-09-18 Thread Bill Bejeck
Hi Bhavesh, I don't see the log file attachment, can you resend it? Also, what version of Kafka Streams are you running? Thanks, Bill On Tue, Sep 18, 2018 at 12:31 PM Bhavesh Patel wrote: > Hi, > > We have a stream application where we are facing a little strange problem > with deleting state

Re: Kafka stream issue : Deleting obsolete state directory

2018-09-18 Thread Bill Bejeck
rror stack attached. > > We are using kafka 2.0. > > Thanks, > Bhavesh Patel > > On Tue, Sep 18, 2018 at 8:04 PM Bill Bejeck wrote: > >> Hi Bhavesh, >> >> I don't see the log file attachment, can you resend it? >> >> Also, what version of Kafk

Re: Kafka stream issue : Deleting obsolete state directory

2018-09-20 Thread Bill Bejeck
; > > > at > > > java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.access$301(ScheduledThreadPoolExecutor.java:180) > > [na:1.8.0_161] > > > > at > > > java.util.concurrent.ScheduledThreadPoolExecutor$Schedul

Re: Terminate Streams application from within Transformer?

2018-09-21 Thread Bill Bejeck
Hi Tim, I wouldn't recommend System.exit(), as it won't give streams a chance to go through a shutdown process. You can throw a RuntimeException from inside the Transformer.transform() method and I will bubble up and the Steams application will shut down. -Bill On Fri, Sep 21, 2018 at 6:59 AM Ti

Re: Referential transparency issue with transform()

2018-09-24 Thread Bill Bejeck
Hi Stephane, Yes, you'll want to have your TransformerSupplier to return a new instance of your Transformer Otherwise, the same Transformer instance is used across all tasks. Since the Transformer can potentially perform stateful operations with a state store, and/or you can schedule punctations u

Re: Kafka stream issue : Deleting obsolete state directory

2018-09-24 Thread Bill Bejeck
gt; *to a large value (Long.MAX_VALUE ?) Thanks, Bill On Thu, Sep 20, 2018 at 4:56 PM Bhavesh Patel wrote: > Hi Bill, > > Yes and thanks for your response. > > Many thanks, > Bhavesh Patel > > On Thu, Sep 20, 2018, 21:31 Bill Bejeck wrote: > > > Hi Bhavesh, > > >

Re: Referential transparency issue with transform()

2018-09-24 Thread Bill Bejeck
Hi Stéphane, Can you provide log files from the crash so we can see the exact failure? Thanks, Bill On Mon, Sep 24, 2018 at 8:12 AM Stéphane. D. wrote: > Hi, > > We just stumbled upon an issue with KStream.transform() where we had a > runtime error with this code: > > ``` > DeduplicationTransf

Re: [KafkaStreams 1.1.1] partition assignment broken?

2018-10-08 Thread Bill Bejeck
Hi Bart, This is a known issue discovered in version 1.1 - https://issues.apache.org/jira/browse/KAFKA-7144 This issue has been fixed in Kafka Streams 2.0, any chance you can upgrade to 2.0? Thanks, Bill On Mon, Oct 8, 2018 at 2:46 PM Bart Vercammen wrote: > Thanks John, > > I'll see what I c

Re: [KafkaStreams 1.1.1] partition assignment broken?

2018-10-09 Thread Bill Bejeck
option. > > In the mean time I have some unit-tests that reproduce this problem, so the > backport to v1.1.1 can easily be verified. > > Greets, > Bart > > On Tue, Oct 9, 2018 at 12:27 AM Bill Bejeck wrote: > > > Hi Bart, > > > > This is a known issue

Re: [ANNOUNCE] New Committer: Manikumar Reddy

2018-10-11 Thread Bill Bejeck
Congrats, Manikumar! On Thu, Oct 11, 2018 at 2:05 PM Rajini Sivaram wrote: > Congratulations, Manikumar! > > On Thu, Oct 11, 2018 at 6:57 PM Suman B N wrote: > > > Congratulations Manikumar! > > > > On Thu, Oct 11, 2018 at 11:09 PM Jason Gustafson > > wrote: > > > > > Hi all, > > > > > > The P

Re: [KafkaStreams 1.1.1] partition assignment broken?

2018-10-15 Thread Bill Bejeck
rt > > On Tue, Oct 9, 2018 at 4:49 PM Bill Bejeck wrote: > > > Hi Bart, > > > > Sounds good. Let me know how it goes. > > > > -Bill > > > > On Tue, Oct 9, 2018 at 5:08 AM Bart Vercammen wrote: > > > > > Hi Bill, > > > >

Re: RocksDB not closed on error during CachingKeyValueStore.flush?

2018-10-23 Thread Bill Bejeck
Hi Patrik, Thanks for reporting this, it does indeed look like this could be an issue. I've created https://issues.apache.org/jira/browse/KAFKA-7534 to look into the issue and if required create a patch. -Bill On Tue, Oct 23, 2018 at 9:24 AM Patrik Kleindl wrote: > Hello > > Can someone pleas

Re: Using GlobalKTable/KeyValueStore for topic cache

2018-11-13 Thread Bill Bejeck
Hi Chris, I'm not sure I totally understand your requirements but the StateRestoreListener ( https://kafka.apache.org/20/javadoc/org/apache/kafka/streams/processor/StateRestoreListener.html) class provides callbacks when restoring state stores (including GlobalStores) and may provide what you are

Re: Using GlobalKTable/KeyValueStore for topic cache

2018-11-13 Thread Bill Bejeck
state store would execute this > "restore state store" operation, even creating a new state store, is that > correct? If so that indeed could be just what I need. > > > On Tue, Nov 13, 2018 at 6:38 PM Bill Bejeck wrote: > > > Hi Chris, > > > > I'

Re: UNKNOWN_PRODUCER_ID error when running Streams WordCount demowith processing.guarantee set to EXACTLY_ONCE

2018-11-21 Thread Bill Bejeck
> Hello Guozhang, Bill > > > > Thanks much for your explanation. I assigned this JIRA issue to myself > > and will > > try to fix it. > > > > Regards, > > Lambdaliu(Shaobo Liu) > > > > On 2018-07-21 at 02:24 Bill Bejeck wrote: > > > > &g

Re: Why do the offsets of the consumer-group (app-id) of my Kafka Streams Application get reset after application restart?

2019-01-14 Thread Bill Bejeck
Hi Jonathan, With EOS enabled, Kafka Streams does not use checkpoint files for restoring state stores; it will replay the data contained in the changelog topic. But this should not affect where the input source topic(s) after a restart also the changelog topics are only consumed from during a rest

Re: [ANNOUNCE] New Committer: Vahid Hashemian

2019-01-15 Thread Bill Bejeck
Congrats! On Tue, Jan 15, 2019 at 5:50 PM Rajini Sivaram wrote: > Congratulations, Vahid! Well deserved!! > > Regards, > > Rajini > > On Tue, Jan 15, 2019 at 10:45 PM Jason Gustafson > wrote: > > > Hi All, > > > > The PMC for Apache Kafka has invited Vahid Hashemian as a project > > committer a

Re: kstream transform forward to different topics

2019-02-07 Thread Bill Bejeck
Hi Nan, To forward to the 3 different topics it will probably be easier to do this in the Processor API. Based off what your stated in your question, the topology will look something like this: final Topology topology = new Topology(); topology.addSource("source-node", "input-topic"); topology.a

Re: kstream transform forward to different topics

2019-02-07 Thread Bill Bejeck
is there any > performance implication? Not sure if I am right, but sounds to me that will > cause one more hop from client(stream app) to kakfa brokers. beginning DSL > processing is happening on the client side. then have to put the result > back to broker, then read back to clien

Re: kstream transform forward to different topics

2019-02-07 Thread Bill Bejeck
. Thanks, Bill On Thu, Feb 7, 2019 at 10:47 AM Bill Bejeck wrote: > Hi Nan, > > What I'm suggesting is do the entire topology in the PAPI, sorry if I > didn't make this clear from before. > > Thanks, > Bill > > On Thu, Feb 7, 2019 at 10:41 AM Nan Xu wrote: > &

Re: kstream transform forward to different topics

2019-02-07 Thread Bill Bejeck
eam = inputStream.transform(transformer) > kstream.map((r1,r2,r3) => r1).to("topic1") > kstream.map((r1,r2,r3) => r2).to("topic2") > kstream.map((r1,r2,r3) => r3).to("topic3") > but don't know if it is the recommended way. > > Thanks, &

Re: Kafka Streams KGroupedTable.count() method returning negative values.

2019-02-08 Thread Bill Bejeck
Hi Ankur, Could you add an additional peek method logging what's going into the groupBy call and share the logs? Thanks, Bill On Fri, Feb 8, 2019 at 7:48 AM Ankur Rana wrote: > Hi All, > > One of my Kafka streams application is returning negative values for > count() method. > How is that poss

Re: Kafka Streams KGroupedTable.count() method returning negative values.

2019-02-08 Thread Bill Bejeck
t; > Also, the negative count is not limited to smaller values, but I have seen > values as high as -300. > I cannot get my head around this behavior. > > > On Fri, Feb 8, 2019 at 5:56 PM Bill Bejeck wrote: > > > Hi Ankur, > > > > Could you add an additional

Re: Kafka Streams & Distributed state question

2019-02-13 Thread Bill Bejeck
Hi Andrian, There is an existing framework for Interactive Queries contributed by Lightbend - https://github.com/lightbend/kafka-streams-query HTH, Bill On Wed, Feb 13, 2019 at 10:14 AM Ryanne Dolan wrote: > Andrian, this looks useful: > > > https://docs.confluent.io/current/streams/develope

Re: [ANNOUNCE] New Committer: Randall Hauch

2019-02-14 Thread Bill Bejeck
Congrats Randall! -Bill On Thu, Feb 14, 2019 at 9:16 PM Guozhang Wang wrote: > Hello all, > > The PMC of Apache Kafka is happy to announce another new committer joining > the project today: we have invited Randall Hauch as a project committer and > he has accepted. > > Randall has been particip

Re: kafka stream. streams.allMetadata() get nothing back.

2019-02-28 Thread Bill Bejeck
Hi Nan, You'll need to set the "application.server" property for Kafka Streams. Since you are using the "streams#allMetadata()" method I'm assuming you are trying to set up interactive queries, so here are two links that should help you get started: https://docs.confluent.io/current/streams/devel

Re: Kafka Streams Disk Usage on upgrade to 2.1.0

2019-03-05 Thread Bill Bejeck
Hi Adrian, No, it's not an expected outcome. Could you file a Jira ticket and include the information requested by Guozhang (code and configs) and we can try to reproduce the error? Thanks, Bill On Tue, Mar 5, 2019 at 10:14 AM Adrian McCague wrote: > Drilling down further: > > bash-4.2# pwd >

Re: [VOTE] 2.2.0 RC2

2019-03-20 Thread Bill Bejeck
+1 (non-binding) - Validated signature - build from source and ran tests (all passed) - verified the quick start and Kafka Streams quick start Thanks for running the release Matthias! -Bill On Tue, Mar 19, 2019 at 7:38 PM Satish Duggana wrote: > +1 (non-binding) > > - Ran testAll/releaseTar

Re: ClassCastException when trying to groupBy after suppress

2019-04-08 Thread Bill Bejeck
Jose, Thanks for reporting this we have created this Jira https://issues.apache.org/jira/browse/KAFKA-8199 for tracking the issue. -Bill On Mon, Apr 8, 2019 at 5:49 AM Jose Lopez wrote: > Hi all, > > My topology is: > > stream -> groupByKey -> windowedBy -> count -> suppress -> groupBy -> > ag

Re: ClassCastException when trying to groupBy after suppress

2019-04-08 Thread Bill Bejeck
019 at 11:52 AM Jose Lopez wrote: > Thank you for the prompt response, Bill. > > Do you have an estimated date as to when this could be fixed? Are you open > to new developers / collaborators? > > Regards, > Jose > > On Mon, 8 Apr 2019 at 17:06, Bill Bejeck wrote: > &

Re: ClassCastException when trying to groupBy after suppress

2019-04-08 Thread Bill Bejeck
or? > > Regards, > Jose > > On Mon, 8 Apr 2019 at 18:28, Bill Bejeck wrote: > > > Jose, > > > > If you'd like to take a swing at fixing this, by all means, assign the > Jira > > ticket to yourself and submit a PR. > > > > If you don't ha

Re: ClassCastException when trying to groupBy after suppress

2019-04-09 Thread Bill Bejeck
On Mon, 8 Apr 2019 at 23:24, Bill Bejeck wrote: > > > Jose, > > > > You're all set. I look forward to your PR. > > > > Thanks, > > Bill > > > > On Mon, Apr 8, 2019 at 4:59 PM Jose Lopez > > wrote: > > > > > Bill,

Re: Kafka Streams - Java StreamException - Serdes issue

2019-04-09 Thread Bill Bejeck
Hi Gioacchino, If I'm understanding your topology correctly it looks like you are doing a reduce operation where the result is a double. For stateful operations, Kafka Streams uses persistent state stores for keeping track of the update stream. When using the KGroupedStream#reduce method, if you

Re: ClassCastException when trying to groupBy after suppress

2019-04-09 Thread Bill Bejeck
👍 On Tue, Apr 9, 2019 at 4:32 PM Jose Lopez wrote: > Bill, > > Fixed, thank you! > > Regards, > Jose > > On Tue, 9 Apr 2019 at 15:55, Bill Bejeck wrote: > > > Jose, > > > > I can see you listed as a contributor, can you try logging out and &

Re: Partition Strategy in Kafka Stream

2019-04-10 Thread Bill Bejeck
Hi Neeraj, I have a couple of questions, are you trying to adjust the partition assignment of records consumed by streams? In that case, you can assign a custom partition assignor for the producer sending records to the topic feeding the streams application. If you are trying to do custom partit

Re: [ANNOUNCE] New Kafka PMC member: Sriharsh Chintalapan

2019-04-18 Thread Bill Bejeck
Congrats Harsh! -Bill On Thu, Apr 18, 2019 at 3:14 PM Guozhang Wang wrote: > Congrats Harsh! > > > Guozhang > > On Thu, Apr 18, 2019 at 11:46 AM Jun Rao wrote: > > > Hi, Everyone, > > > > Sriharsh Chintalapan has been active in the Kafka community since he > became > > a Kafka committer in 201

Re: [ANNOUNCE] New Kafka PMC member: Matthias J. Sax

2019-04-18 Thread Bill Bejeck
Congrats Matthias! Well deserved! -Bill On Thu, Apr 18, 2019 at 5:35 PM Guozhang Wang wrote: > Hello Everyone, > > I'm glad to announce that Matthias J. Sax is now a member of Kafka PMC. > > Matthias has been a committer since Jan. 2018, and since then he continued > to be active in the communi

Re: Please give me permission to create KIP,Thanks!

2019-05-13 Thread Bill Bejeck
Thanks for your interest in contributing! You should be all set now. Thanks, Bill On Mon, May 13, 2019 at 12:34 PM hu xiaohua wrote: > > > > I want create my KIP ,but I havn't a permission,please give me permission > to create KIP,My Wiki ID is Flowermin,thanks

Re: org.apache.kafka.streams.processor.TimestampExtractor#extract method in version 2.3 always returns -1 as value

2019-06-26 Thread Bill Bejeck
Hi Jonathan, Thanks for reporting this. Which timestamp extractor are you using in the configs? Thanks, Bill On Wed, Jun 26, 2019 at 9:14 AM Jonathan Santilli < jonathansanti...@gmail.com> wrote: > Hello, hope you all are doing well, > > am testing the new version 2.3 for Kafka Streams specifi

Re: org.apache.kafka.streams.processor.TimestampExtractor#extract method in version 2.3 always returns -1 as value

2019-06-26 Thread Bill Bejeck
eers! > -- > Jonathan > > > On Wed, Jun 26, 2019 at 3:16 PM Bill Bejeck wrote: > > > Hi Jonathan, > > > > Thanks for reporting this. Which timestamp extractor are you using in > the > > configs? > > > > Thanks, > > Bill > > >

Re: org.apache.kafka.streams.processor.TimestampExtractor#extract method in version 2.3 always returns -1 as value

2019-06-28 Thread Bill Bejeck
> >> -Bill > >> > >> > >> On Wed, Jun 26, 2019 at 10:23 AM Jonathan Santilli < > >> jonathansanti...@gmail.com> wrote: > >> > >>> Hello Bill, > >>> > >>> am implementing the TimestampExtractor Interface, then using it to > &

Re: Kafka Streams not fetching data from one partition

2019-07-11 Thread Bill Bejeck
Hi Piotr, Thanks for reporting this issue. Can you provide full kafka-streams and broker logs around the timeframe you observed this? -Bill On Thu, Jul 11, 2019 at 8:53 AM Piotr Strąk wrote: > Hello, > > I'm investigating an issue in which a Kafka Streams application does not > consume from o

Re: kafka-streams disconnection errors

2019-07-11 Thread Bill Bejeck
Hi Javier, Your theory could be correct, but it's hard to say exactly without looking at some more information. Can you provide your streams configuration and logs (both streams and broker). Thanks, Bill On Thu, Jul 11, 2019 at 2:55 AM Javier Arias Losada wrote: > Hello there, > > I managed to

Re: Facing memory issues with kafka streams application

2019-07-11 Thread Bill Bejeck
Thanks for reporting this Kalyani, we'll take a look. By chance can provide log files? Thanks, Bill On Mon, Jul 8, 2019 at 7:43 AM kalyani yarlagadda < kalyani.yarlagad...@gmail.com> wrote: > Hi, > > I need assistance in the below scenario. Please help me with this. > > I am using the hopping ti

Re: Kafka streams (2.1.1) - org.rocksdb.RocksDBException:Too many open files

2019-07-12 Thread Bill Bejeck
Hi Thameem, > Our topology has 3100 tasks with 3 brokers and 8 partitions. I have a question that is somewhat orthogonal to your original item. I may be missing something, but are you saying your streams application has 3,100 tasks from 8 input partitions? If that is the case, do you mind sharin

  1   2   >