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

2019-07-02 Thread Sophie Blee-Goldman
This can also happen if you have any open iterators that you forget to close (for example using IQ), although that's probably not what's going on here since 3152 is certainly a lot of rocks instances for a single fs. There's no default number of open files per instance, since rocks creates new fil

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

2019-07-02 Thread Sophie Blee-Goldman
It sounds like rocksdb *is* honoring your configs -- the max.open.files config is an internal restriction that tells rocksdb how many open files it is allowed to have, so if that's set to -1 (infinite) it won't ever try to limit its open files and you may hit the OS limit. Think of it this way: if

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

2019-07-02 Thread Sophie Blee-Goldman
Tue, Jul 2, 2019 at 11:10 PM emailtokir...@gmail.com < emailtokir...@gmail.com> wrote: > > > On 2019/07/03 05:46:45, Sophie Blee-Goldman wrote: > > It sounds like rocksdb *is* honoring your configs -- the max.open.files > > config is an internal restriction that tells ro

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

2019-07-03 Thread Sophie Blee-Goldman
ere is no abnormality with the threads / memory or > heap. > > > > Thanks > > Thameem > > > >> On Jul 3, 2019, at 11:50 AM, Sophie Blee-Goldman > wrote: > >> > >> How sure are you that the open file count never goes beyond 50K? Are > those > &

Re: stream.filter() based on message header

2019-07-08 Thread Sophie Blee-Goldman
Hi Jorg, The transform/transformValues/process operators are not inherently stateful. They *can* be stateful if you choose to connect a statestore, but otherwise they are stateless and in your case, most likely what you want to use. Cheers, Sophie On Thu, Jul 4, 2019 at 2:51 AM Jorg Heymans wro

Re: Kafka 2.3.0 - inMemoryKeyValueStore changes (KAFKA-7918) - java.lang.NullPointerException

2019-07-08 Thread Sophie Blee-Goldman
Hi Nitay, InMemoryKeyValueStore is in the internal package, not part of the public API, so it is not meant to be used directly since it's implementation may change at any time (as has happened here). It's intended that you use the store builders/suppliers to get a new state store, but as you notic

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

2019-07-08 Thread Sophie Blee-Goldman
Thanks for the notice Jonathan! We tracked down the problem and it should be an easy fix: https://github.com/apache/kafka/pull/6719/files On Fri, Jul 5, 2019 at 6:25 AM Jonathan Santilli wrote: > Thanks a lot Bill for creating the issue, I have updated it with a little > bit more of info. > > Ch

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

2019-07-11 Thread Sophie Blee-Goldman
I believe the "resource temporarily unavailable" actually is related to the open files, most likely you are hitting the total file descriptor limit. Sorry if you mentioned this and I missed it, but what was the max.open.files in your RocksDBConfigSetter when you ran this? Actually, could you just i

Re: Kafka Streams - unbounded memory growth - stateful processing (rocksdb)

2019-07-16 Thread Sophie Blee-Goldman
Hi Ashok, 1) RocksDB uses memory in four ways, one of which (iterators) *should* be negligible -- however if you have a very large number of them open at any one time, they can consume a lot of memory (until they are closed). If you are opening many iterators throughout the day, consider closing t

Re: Kafka Streams - unbounded memory growth

2019-07-17 Thread Sophie Blee-Goldman
If you are using inMemoryKeyValueStore, the records are stored by definition in memory. RocksDB is not used at all. This store will continue to grow proportionally to your keyspace. If you do not have sufficient memory to hold your entire dataset in memory, consider adding another instance or switc

Re: Kafka Streams - unbounded memory growth

2019-07-17 Thread Sophie Blee-Goldman
Sorry, didn't see the "off-heap" part of the email. Are you using any stateful DSL operators? The default stores are persistent, so you may have a RocksDB store in your topology without explicitly using one. On Wed, Jul 17, 2019 at 10:12 AM Sophie Blee-Goldman wrote: >

Re: Kafka Streams - unbounded memory growth

2019-07-17 Thread Sophie Blee-Goldman
> } > > > and included in the stream config .. > > settings.put(StreamsConfig.ROCKSDB_CONFIG_SETTER_CLASS_CONFIG, > classOf[CustomRocksDBConfig]) > > > Regards > Ashik > > > > > > > > > > On Wed, Jul 17, 2019 at 10:52 PM Sophie Blee

Re: Kafka Streams - unbounded memory growth

2019-07-17 Thread Sophie Blee-Goldman
; > > } > > > } > > > > > > > > > and included in the stream config .. > > > > > > settings.put(StreamsConfig.ROCKSDB_CONFIG_SETTER_CLASS_CONFIG, > > > classOf[CustomRocksDBConfig]) > > > > > > > > > Regards > > > Ashik > >

Re: Kafka Streams - unbounded memory growth

2019-07-17 Thread Sophie Blee-Goldman
SeWlanStatsAggregator/1_4 > 4.0K ./buzzard.MonitoringSeWlanStatsAggregator/0_1 > 4.0K ./buzzard.MonitoringSeWlanStatsAggregator/0_4 > 28K ./buzzard.MonitoringSeWlanStatsAggregator > 32K . > > > > On Wed, Jul 17, 2019 at 11:18 PM Sophie Blee-Goldman > wrote: > > > Hm

Re: Kafka Streams - unbounded memory growth

2019-07-18 Thread Sophie Blee-Goldman
onSource > Sink: StationSink (topic: > buzzard.app.monitoring.internal.clientsestats) > <-- StationProcess > > Regards > Ashik > > On Thu, Jul 18, 2019 at 1:31 AM Sophie Blee-Goldman > wrote: > > > Hm. These directories shouldn't be created if using only an in memory &

Re: Kafka Streams - unbounded memory growth

2019-07-19 Thread Sophie Blee-Goldman
wrote: > Hi If I’m not wrong I remember seeing the streams code ., that default > rocksdb state.dir is rocksdb itself. Any content would go under > /tmp/Kafka-streams/rocksdb > > On Fri, 19 Jul 2019 at 1:55 AM, Sophie Blee-Goldman > wrote: > > > And all four stores (Bucket

Re: Kafka Streams and broker compatibility

2019-08-26 Thread Sophie Blee-Goldman
I'm pretty sure one of the Suppress bug fixes that went into 2.2.1 involved adding headers. Updating the compatibility matrix must have just slipped when that bugfix was merged -- thanks for bringing this up! On Mon, Aug 26, 2019 at 5:37 PM Alisson Sales wrote: > Hi Guozhang, thanks for your rep

Re: Stale data in KStream->KTable join

2019-10-04 Thread Sophie Blee-Goldman
I think the issue here is that you're basically creating a cycle in your streams topology, which is generally supposed to be a DAG. If I understand correctly, rather than writing the new data to the underlying store you're sending it to the topic from which the table is built. Is that right? The p

Re: Stream branching and states

2019-10-06 Thread Sophie Blee-Goldman
Hi Navneeth, I think your plan to implement branching with the Processor API sounds reasonable, but I'm wondering why not just use the DSL branch operator and connect some transformers? That will let you mix the DSL and PAPI so you don't have to re-implement anything that is already in the DSL. I

Re: Long delay between incoming and outgoing messages using kafka streams

2019-10-09 Thread Sophie Blee-Goldman
Hi Petter, I'd recommend turning off caching by setting p.put(StreamsConfig.CACHE_MAX_BYTES_BUFFERING, 0); 2.3.0 also has some known performance issues that will be fixed in 2.3.1, but they shouldn't be noticeable if you turn caching off and aren't reading/writing to topics with a very high parti

Re: Stale data in KStream->KTable join

2019-10-14 Thread Sophie Blee-Goldman
t; the KStream to KStream>. The individual > > messages on that topic will be larger, but lower frequency. But that > won't > > solve the problem of replaying from earlier offsets - which effectively > > will be another bursty scenario. > > > > If that doesn&#

Re: Very low bytesout for internal repartition topic, is it related to RocksDB?

2019-10-14 Thread Sophie Blee-Goldman
Out of curiosity, which version are you using? There's nothing that really jumps out at me as problematic in your RocksDBConfigSetter, but note that I think you may need to increase the number of threads in the "LOW priority thread pool" in addition to setting the maxBackgroundCompactions -- this

Re: Long delay between incoming and outgoing messages using kafka streams

2019-10-14 Thread Sophie Blee-Goldman
the latency of 1s per processing step by flushing buffers every second. > Is this the case or does these two configuration values interact in some > other way? > > We are using 256 partitions for all our topics. Is this to be considered a > very high partition count? Do you think

Re: Very low bytesout for internal repartition topic, is it related to RocksDB?

2019-10-14 Thread Sophie Blee-Goldman
with(Serdes.String(), new MessagetSerde())) > .reduce((value1, value2) -> { > long count1 = value1.getCount(); > long count2 = value2.getCount(); > value2.setCount(count1 + count2); > return value2; > } > ) > .toStream() > .selectKey( ... ) > .t

Re: Very low bytesout for internal repartition topic, is it related to RocksDB?

2019-10-14 Thread Sophie Blee-Goldman
o.IOException: Broken pipe). Is is also related to the rocksDB > read and write? Anything I should do to get rid of this exception? > > Thanks a lot! > > On Mon, Oct 14, 2019 at 6:10 PM Sophie Blee-Goldman > wrote: > > > > Ah ok, 2.3.0 has a known performance issue

Re: Very low bytesout for internal repartition topic, is it related to RocksDB?

2019-10-22 Thread Sophie Blee-Goldman
imeout issue? Does > it mean it takes too long to send the messages to this changelog > topic? > > Thanks > Kathy > > On Mon, Oct 14, 2019 at 10:54 PM Sophie Blee-Goldman > wrote: > > > > Glad that helped! > > > > Honestly I can't say I re

Re: Reducing streams startup bandwidth usage

2019-12-09 Thread Sophie Blee-Goldman
It might be that the cache appears to "stop working" because it gets full, and each new update causes an eviction (of some older record). This would also explain the opposite behavior, that it "starts working" again after some time without being restarted, since the cache is completely flushed on c

Re: Reducing streams startup bandwidth usage

2019-12-09 Thread Sophie Blee-Goldman
rds go to the changelog topic > until the old ones are evicted? > > Regarding the timing, what timing do you mean? Between when the cache stops > and starts working again? We're using EOS os I believe the commit interval > is every 100ms. > > Regards > > -- > Ale

Re: Reducing streams startup bandwidth usage

2019-12-09 Thread Sophie Blee-Goldman
so generated by simulators with very predictable > output rate. > > In the meantime I've enabled reporting of debug metrics (so including cache > hit ratio) to hopefully get better insights the next time it happens. > > Thank you in advance > > -- > Alessandro T

Re: Reducing streams startup bandwidth usage

2019-12-12 Thread Sophie Blee-Goldman
r key in the store. > > > > The only thing I don't understand is why cache works 80% of the time and > > then suddenly the changelog sent bytes increase 90x. > > I mean, if cache wasn't working, why enabling it in our pipeline > decreased > > the sent bytes

Re: "Skipping record for expired segment" in InMemoryWindowStore

2020-02-10 Thread Sophie Blee-Goldman
While I agree that seems like it was probably a refactoring mistake, I'm not convinced it isn't the right thing to do. John, can you reiterate the argument for setting it to debug way back when? I would actually present this exact situation as an argument for keeping it as warn, since something in

Re: "Skipping record for expired segment" in InMemoryWindowStore

2020-02-11 Thread Sophie Blee-Goldman
le skipped-record > metric > > is non-zero. And then you may not even observe it again. > > > > I either missed the memo on that discussion, or participated in it and > then > > forgot it even happened. I'm not sure I want to look back at the thread > to > > find ou

Re: Kafka Streams - partition assignment for the input topic

2020-03-20 Thread Sophie Blee-Goldman
Although it's not the main objective, one side effect of KIP-441 should be improved balance of the final stable assignment. By warming up standbys before switching them over to active tasks we can achieve stickiness without sacrificing balance in the followup rebalance. This work is targeted for t

Re: Kafka Streams - partition assignment for the input topic

2020-03-23 Thread Sophie Blee-Goldman
to > send deletes and updates to records in the topic (by nullifying keys etc) > and also so my streams app automatically recalculates various aggregations > in all of the sub-topologies. Could this be the cause of the problem? > > Stephen > > On Fri, 20 Mar 2020 at 17:33,

Re: Partition assignment not well distributed over threads

2020-07-29 Thread Sophie Blee-Goldman
Hey Giselle, How many stream threads is each instance configured with? If the total number of threads across all instances exceeds the total number of tasks, then some threads won't get any assigned tasks. There's a known bug where tasks might not get evenly distributed over all instances in this

Re: JNI linker issue on ARM (Raspberry PI)

2020-08-24 Thread Sophie Blee-Goldman
Yeah, if you weren't already running it, try upgrading Streams to 2.6. They recently added support for "all" platforms to rocksdb and we updated the dependency to get this fix in 2.6. See KAFKA-9225 If you already were running 2.6, then, that's un

Re: Streams constantly reblancing,

2020-09-02 Thread Sophie Blee-Goldman
Hey steve, I guess the first question I have is, is the consumer group actually rebalancing or does KafkaStreams just appear to be "stuck" in the REBALANCING state? If this application has been run before and there is a large amount of data in the changelog topics, it might take a while to rebuild

Re: Kafka stream error - Consumer is not subscribed to any topics or assigned any partitions

2020-09-11 Thread Sophie Blee-Goldman
You should upgrade to 2.5.1, it contains a fix for this. Technically the "fix" is just to automatically set the num.stream.threads to 0 when a global-only topology is detected, so setting this manually would accomplish the same thing. But the fix also includes a tweak of the KafkaStreams state mac

Re: Streams constantly reblancing,

2020-09-11 Thread Sophie Blee-Goldman
ile (filter.log) shows the logs for about an hour and a > quarter run, my confusion comes from the fact that looking via JMX and at > the CPU logs it appears that nothing much is actually being done on the > Kafka side but there aren't any exceptions. > > Help and hints appreciated

Re: Kafka stream error - Consumer is not subscribed to any topics or assigned any partitions

2020-09-14 Thread Sophie Blee-Goldman
gy. In our application that is what we are doing > and > > > there is no source topic for the stream to process data from, i mean > there > > > is however it is done through a consumer-producer kind of design and > not > > > through stream topology. > &g

Re: Stream stopped running due to TimeoutException: Timeout expired after 60000milliseconds while awaiting InitProducerId

2020-10-19 Thread Sophie Blee-Goldman
Hey Pushkar, If I remember correctly there were a number of broker-side bugs that were uncovered over the course of this ticket, any of which could result in the exception you've seen. I'm *pretty* sure that they should all be fixed in 2.5.0, but it's possible that some of the fixes only made it i

Re: Kafka Streams RocksDB CPU usage

2020-10-27 Thread Sophie Blee-Goldman
You might want to start with a lower commit interval, if you can handle some additional latency. I would bet that the frequent flushing is a major part of your problem: not just the act of flushing itself, but the consequences for the structure of the data in each rocksdb. If you end up flushing un

Re: Kafka Streams application stuck rebalancing on startup

2020-10-27 Thread Sophie Blee-Goldman
> > We've been able to get the crucial factors that cause this behavior down to > a particular combination What do you mean by this -- that you only see this when all four of those operators are at play? Or do you see it with any of them. I guess the first thing to narrow down is whether it's act

Re: Kafka Streams application stuck rebalancing on startup

2020-10-28 Thread Sophie Blee-Goldman
nski/streams-issue-demo/blob/master/src/main/java/com/github/ajablonski/StreamsConfiguration.java> > is the configuration and this > <https://github.com/ajablonski/streams-issue-demo/blob/master/src/test/java/com/github/ajablonski/StreamsConfigurationTest.java> > is the test. &g

Re: Kafka Streams application stuck rebalancing on startup

2020-11-02 Thread Sophie Blee-Goldman
't alter the endless-rebalancing > behavior. > > We also see the endless rebalancing behavior in a real Kafka cluster, > using input and output topics that have already been created (and are > readily consumed from and written to). > > > > > On Wed, Oct 28, 2020 a

Re: Kafka Streams application stuck rebalancing on startup

2020-11-05 Thread Sophie Blee-Goldman
some thoughts on the JIRA ticket and will try to incorporate one of them into the fix as well. On Mon, Nov 2, 2020 at 9:38 PM Sophie Blee-Goldman wrote: > Ok I tried to reproduce the issue with a minimal example, and saw the same > results. > It seems like there's something weird

Re: Guidance to downgrade the stateful Kafka stream application?

2020-12-17 Thread Sophie Blee-Goldman
Hey Ming, There should not be any issues in downgrading from 2.5 to 2.2, and if you stayed on the eager protocol then you can do the downgrade in a single rolling bounce. It sounds like your main concern here is with the RocksDB version bump, and whether there would be any problems reading/openin

Re: [VOTE] 2.7.0 RC6

2020-12-17 Thread Sophie Blee-Goldman
Thanks for driving this release! I built from the tag and ran the tests, and verified the signatures. +1 (binding) Sophie On Thu, Dec 17, 2020 at 3:53 PM Jakub Scholz wrote: > +1 (non-binding) ... I used the binaries (Scala 2.12) and the staged Maven > artifacts - all seems to work fine. Thank

Re: Create a new consumer for a consumer group from within the assign method of the rebalancing interface (AbstractPartitionAssignor)

2021-02-25 Thread Sophie Blee-Goldman
If you create a new consumer inside the assign method (and assuming you actually start polling with it so that it sends the JoinGroup request), then yes, it would need a new rebalance to accommodate this consumer. The group coordinator will inform all the existing members to rejoin the group so tha

Re: Rebalancing stuck, never finishes

2021-02-25 Thread Sophie Blee-Goldman
Hey Peter, It does sound like you may have hit https://issues.apache.org/jira/browse/KAFKA-9752 You will need to upgrade your brokers in order to get the fix, since it's a broker-side issue On Tue, Feb 9, 2021 at 2:48 AM Péter Sinóros-Szabó wrote: > Hi, > > I have an application running with 6

Re: Rebalancing stuck, never finishes

2021-02-26 Thread Sophie Blee-Goldman
Peter, It does seem like KAFKA-9752 is the most likely suspect, although if your clients were upgraded to 2.6.1 then I don't believe they would be on an early enough version of the JoinGroup to run into this. I'm not 100% sure though, it may be a good idea to leave a comment on that ticket and pin

[VOTE] 2.6.2 RC0

2021-03-12 Thread Sophie Blee-Goldman
Hello Kafka users, developers and client-developers, This is the first candidate for release of Apache Kafka 2.6.2. Apache Kafka 2.6.2 is a bugfix release and fixes 30 issues since the 2.6.1 release. Please see the release notes for more information. Release notes for the 2.6.2 release: https://

Re: Rebalancing and scaling of consumers on kubernetes, instanteous scale to x consumer replicas ==> x rebalancing?

2021-03-15 Thread Sophie Blee-Goldman
Hey Mazen, There's not necessarily one rebalance per new consumer, in theory if all 100 consumers are started up at the same time then there may be just a single rebalance. It really depends on the timing -- for example in the log snippet you provided, you can see that the first member joined at 1

Re: Redis as state store

2021-03-15 Thread Sophie Blee-Goldman
This certainly does seem like a flaw in the Streams API, although of course Streams is just in general not designed for use with remote anything (API calls, stores, etc) That said, I don't see any reason why we *couldn't* have better support for remote state stores. Note that there's currently no

Re: Redis as state store

2021-03-15 Thread Sophie Blee-Goldman
spend some time digging into > more of how this works first, but will then try to gather my thoughts and > get something created. > > Alex > > On Mon, Mar 15, 2021 at 1:48 PM Sophie Blee-Goldman > wrote: > > > This certainly does seem like a flaw in the Streams API,

Re: Redis as state store

2021-03-15 Thread Sophie Blee-Goldman
Yep, that fell off my radar. Here we go: https://issues.apache.org/jira/browse/KAFKA-12475 On Mon, Mar 15, 2021 at 8:09 PM Guozhang Wang wrote: > Hey Sophie, > > Maybe we can first create a JIRA ticket for this? > > On Mon, Mar 15, 2021 at 3:09 PM Sophie Blee-Goldman > w

Re: [VOTE] 2.6.2 RC0

2021-03-15 Thread Sophie Blee-Goldman
Luke > > On Tue, Mar 16, 2021 at 11:12 AM Guozhang Wang wrote: > > > Hi Sophie, > > > > I've reviewed the javadocs / release notes / documentations, and they > LGTM. > > > > +1. > > > > > > Guozhang > > > > On Fri, Mar 12,

Re: Slightly Modified Sticky Assignor.

2021-03-15 Thread Sophie Blee-Goldman
Hey Mazen, The easiest way to approach this is probably to pass in a reference to the associated Consumer and then just call one of the *Consumer#committed *methods which return the OffsetAndMetadata. But I'm guessing your underling question may be about how to get that reference to the Consumer

Re: Kafka custom partition - consumers assignor with custom per partition user/custom data

2021-03-15 Thread Sophie Blee-Goldman
I believe I already answered your question in another channel, but just to follow up in this thread in case anyone else is interested in the answer: You can override the *ConsumerPartitionAssignor.onAssignment(Assignment, ConsumerGroupMetadata)* method to get an update on the currently assigned p

Re: Slightly Modified Sticky Assignor.

2021-03-16 Thread Sophie Blee-Goldman
> > Best, > ________ > From: Sophie Blee-Goldman > Sent: Tuesday, March 16, 2021 5:58 AM > To: users@kafka.apache.org > Subject: Re: Slightly Modified Sticky Assignor. > > Hey Mazen, > > The easiest way to approach this is probably to pass in a

Re: Kafka Streams And Partitioning

2021-03-17 Thread Sophie Blee-Goldman
Hey Gareth, Kafka Streams state store partitioning is based on the partitioning of the upstream input topics. If you want your RocksDB stores to be partitioned based on the prefix of a key, then you should make sure the input topic feeding into it uses whatever partitioning strategy you had in min

Re: Redis as state store

2021-03-19 Thread Sophie Blee-Goldman
case of stream application error? > > On Tue, Mar 16, 2021 at 12:18 AM Sophie Blee-Goldman > wrote: > > > This certainly does seem like a flaw in the Streams API, although of > course > > Streams is just > > in general not designed for use with remote anything (API

Re: [kafka-clients] [VOTE] 2.6.2 RC0

2021-03-19 Thread Sophie Blee-Goldman
jira/browse/KAFKA-12508 > > > > I set its priority to blocker since the bug might break at-least-once > > and exactly-once processing guarantees. > > > > Feel free to set it back to major, if you think that it is not a blocker. > > > > Best, > > Br

Re: [kafka-clients] [VOTE] 2.7.1 RC0

2021-03-19 Thread Sophie Blee-Goldman
Hey Mickael, I just merged the fix back to 2.7 so you should be good to go Thanks for the PR Bruno! On Fri, Mar 19, 2021 at 9:34 AM Mickael Maison wrote: > Thanks Bruno, > > That indeed sounds like a blocker. > > I'm closing this vote, I'll build a new RC once a fix is merged into 2.7 > > On Fr

Re: Kafka Streams And Partitioning

2021-03-19 Thread Sophie Blee-Goldman
> read via REST all values for A. So to make this work I have to have a > repartition topic even though I am not really repartitioning (i.e. all > records for A should still be processed > together). Is my understanding correct? > > So WindowedStreamPartitioner is a special case for avoi

Re: Redis as state store

2021-03-20 Thread Sophie Blee-Goldman
; making the state store inconsistent in first place, thus saving wipe out > and rebuild ? If so then this also doesn't need to halt other stream > applications and would prove much more better approach from performance > point of view. Is that correct? > > On Sat, Mar 20,

Re: Redis as state store

2021-03-23 Thread Sophie Blee-Goldman
ocal state store > and came across that kafka also provides an RPC layer on top of state store > which allows a steam application to query state stored in local state store > of another stream application instance. Is that correct? If so then we can > try that option instead of redis stat

Re: [kafka-clients] [VOTE] 2.6.2 RC0

2021-03-31 Thread Sophie Blee-Goldman
release. > > Kind regards > Georg > > -Original Message- > From: Sophie Blee-Goldman > Sent: Friday, March 19, 2021 11:44 PM > To: Bruno Cadonna > Cc: dev ; users@kafka.apache.org; > kafka-clie...@googlegroups.com > Subject: Re: [kafka-clients] [VOTE] 2.6.2 RC0 &g

Re: [ANNOUNCE] New Committer: Bruno Cadonna

2021-04-07 Thread Sophie Blee-Goldman
Congrats! On Wed, Apr 7, 2021 at 6:32 PM Luke Chen wrote: > Congrats Bruno!! > > Luke > > On Thu, Apr 8, 2021 at 9:18 AM Matthias J. Sax wrote: > > > Congrats Bruno! Very well deserved! > > > > > > -Matthias > > > > On 4/7/21 3:51 PM, Bill Bejeck wrote: > > > Congrats Bruno! Well deserved. > >

[VOTE] 2.6.2 RC1

2021-04-08 Thread Sophie Blee-Goldman
Hello Kafka users, developers and client-developers, This is the second candidate for release of Apache Kafka 2.6.2. Apache Kafka 2.6.2 is a bugfix release and fixes 35 issues since the 2.6.1 release. Please see the release notes for more information. Release notes for the 2.6.2 release: https:/

Re: Deletion a specific consumer of a consumer group through the Kafka admin client API.

2021-04-09 Thread Sophie Blee-Goldman
Hey Mazen, There is technically an AdminClient API which allows you to remove a single member from the group: Admin.removeMembersFromConsumerGroup(String groupId, RemoveMembersFromConsumerGroupOptions options); but I don't think this will help you. For one thing, this API only works for removing

[ANNOUNCE] Apache Kafka 2.6.2

2021-04-21 Thread Sophie Blee-Goldman
, and Zalando, among others. A big thank you for the following 26 contributors to this release! A. Sophie Blee-Goldman, Andy Wilkinson, Bob Barrett, Bruno Cadonna, Chia-Ping Tsai, Chris Egerton, dengziming, Greg Harris, Ismael Juma, Jim Galasyn, John Roesler, Julien Jean Paul Sirocchi, Lee Dongjin

Re: Unable to put more than 3 consumers in a single process

2021-05-08 Thread Sophie Blee-Goldman
Hey Lahiru, As the log message describes, the member is leaving the group because the consumer has been closed by your process/application. You need to track down where in your application code the Consumer#close method is being called, and why. There's no technical limit on how many consumers ca

Re: Kafka contributor list request

2021-05-27 Thread Sophie Blee-Goldman
Done, added you to Confluence and Jira so you should be able to self-assign tickets and create KIPs if necessary. Welcome to Kafka :) On Thu, May 27, 2021 at 4:28 PM Norbert Wojciechowski < wojciechowski.norbert.git...@gmail.com> wrote: > Hello, > > Can I please be assigned to Kafka contributor

Re: Consumer Group Stuck on "Completing-Rebalance" State

2021-06-22 Thread Sophie Blee-Goldman
Hey Tao, We recently discovered a bug in the way that the consumer tracks partition metadata which may cause the cooperative-sticky assignor to throw this exception in the case of a consumer that dropped out of the group at some point. I'm just about to file a ticket for it, and it should be fixed

Re: Consumer Group Stuck on "Completing-Rebalance" State

2021-06-22 Thread Sophie Blee-Goldman
Here's the ticket: https://issues.apache.org/jira/browse/KAFKA-12984 And the root cause of that itself: https://issues.apache.org/jira/browse/KAFKA-12983 On Tue, Jun 22, 2021 at 6:15 PM Sophie Blee-Goldman wrote: > Hey Tao, > > We recently discovered a bug in the way that the c

Re: Consumer Group Stuck on "Completing-Rebalance" State

2021-06-23 Thread Sophie Blee-Goldman
> "at > > most once" event consuming. Does assignor strategy (cooperative-sticky vs > > sticky vs others) really matter in this case? My understanding is that, > > regardless which strategy is used, the members in the group have to > rejoin > > when re-balance h

Re: documentation css

2021-07-12 Thread Sophie Blee-Goldman
I see what you mean, that's pretty ugly. Can you file a bug report at https://issues.apache.org/jira/issues/ so we can track and follow up on this? Thanks! On Mon, Jul 12, 2021 at 7:35 AM Rasto Janotka wrote: > Hi, > I am using your documentation it is written very nice and clean but there > is

Re: Consumer group rebalancing

2021-07-23 Thread Sophie Blee-Goldman
Check out the GroupCoordinator logs, they say why the group is rebalancing. Specifically the one that begins "Preparing to rebalance group" If you search for that in the logs you shared, it shows the reason: 2021-07-22 09:26:37,202 INFO [GroupCoordinator 1]: Preparing to rebalance > group shows_e

Re: Kafka Streams leave group behaviour

2021-08-18 Thread Sophie Blee-Goldman
As Boyang mentioned, Kafka Streams intentionally does not send a LeaveGroup request when shutting down. This is because often the shutdown is not due to a scaling down event but instead some transient closure, such as during a rolling bounce. In cases where the instance is expected to start up agai

Re: Downgrading KafkaStreams

2021-10-14 Thread Sophie Blee-Goldman
Matthias, shouldn't version probing prevent this sort of thing from happening? That is, shouldn't a live downgrade always be possible? I know there were some bugs in the downgrade path for version probing in the past, but I'm pretty sure they should have been fixed by 2.4.1 This seems like a bug t

Re: ConsumerRecord / ProducerRecord common interface?

2021-10-22 Thread Sophie Blee-Goldman
I agree, this has always really annoyed me. This will definitely require a KIP since it's a public feature, but it should (hopefully!) be a fairly quick and non-controversial one. You can read up on the process and get started here

Re: Kafka Streams Topology State

2022-08-18 Thread Sophie Blee-Goldman
Hey Peter Try clearing the local state -- if you have stateful tasks then by default Streams will use rocksdb to store records locally in directories specific to/named after that task. This is presumably why you're seeing errors related to "the task for peek node missing in old nodes" You can del

Re: kafka stream zombie state

2022-08-19 Thread Sophie Blee-Goldman
Well it sounds like your app is getting stuck somewhere in the poll loop so it's unable to call poll again within the session timeout, as the error message indicates -- it's a bit misleading as it says "Sending LeaveGroup request to coordinator" which implies it's *currently* sending the LeaveGroup

Re: Kafka Streams - Producer attempted to produce with an old epoch.

2022-10-27 Thread Sophie Blee-Goldman
I'm not one of the real experts on the Producer and even further from one with broker performance, so someone else may need to chime in for that, but I did have a few questions: What specifically are you unsatisfied with w.r.t the performance? Are you hoping for a higher throughput of your Streams

Re: Cannot send in state COMMITTING_TRANSACTION

2022-11-15 Thread Sophie Blee-Goldman
Interesting, this does look like it could be a bug in Streams and I'm not aware of any known or already-fixed issues resembling this. Could you file a bug report over here and include as much context/information as possible? Providi

Re: Out of order messages when kafka streams application catches up

2022-11-15 Thread Sophie Blee-Goldman
Tomasz you'll need to upgrade the kafka Streams dependency to 3.0 (or above) to get the fix that John mentioned before -- this behavior is known/expected on earlier versions such as 2.8.1 as you are using On Tue, Nov 15, 2022 at 2:21 AM Tomasz Gac wrote: > Hi John, > > I've reviewed the test you

Re: Kafka Streams possible partitioner bug

2022-11-18 Thread Sophie Blee-Goldman
Hey Upesh, are you trying to plug in the custom partitioner via the `partitioner.class` ProducerConfig? That won't work in Streams for the exact reason you highlighted, which is why Streams has its own version of the interface called StreamPartitioner -- this is what you need to implement instead.

[VOTE] 3.4.0 RC1

2023-01-20 Thread Sophie Blee-Goldman
is thread with the successful builds as we get them. /********** Thanks, Sophie Blee-Goldman

Re: GlobalKTable with RocksDB - queries before state RUNNING?

2023-11-21 Thread Sophie Blee-Goldman
Just to make sure I understand the logs, you're saying the "new file processed" lines represent store queries, and presumably the com.osr.serKafkaStreamsService is your service that's issuing these queries? You need to wait for the app to finish restoring state before querying it. Based on this me

Re: GlobalKTable with RocksDB - queries before state RUNNING?

2023-11-26 Thread Sophie Blee-Goldman
al stores do not yet support the KafkaStreams#query API. Use > KafkaStreams#store instead." > > From my point of view it would be great if this will work and behave like > with IN_MEMORY StoreType as it is straight forward to use. > > Do you see a chance to get InteractiveQu

Re: Is the KafkaStreams#store() method thread-safe?

2023-12-26 Thread Sophie Blee-Goldman
Hey Kohei, Good question -- I don't think there's exactly a short answer to this seemingly simple question so bear with me for a second. My understanding is that KafkaStreams#store is very much intended to be thread-safe, and would have been back when it was first added a long time ago, and the j

Re: Is the KafkaStreams#store() method thread-safe?

2024-01-02 Thread Sophie Blee-Goldman
7;m interested in submitting a patch, but I > cannot make any commitment): > https://issues.apache.org/jira/browse/KAFKA-16055 > > Regards, > Kohei > > > > On Dec 27, 2023, at 5:43, Sophie Blee-Goldman > wrote: > > > > Hey Kohei, > > > > Good

Re: Kafka Stream App Rolling Restarts - Too Many Rebalances Per Partition

2024-05-06 Thread Sophie Blee-Goldman
Hey, Just skimming the config list, there are two things that immediately jumped out at me: 1. The default session timeout was bumped up to 45 seconds a little while ago. Not sure if you're overriding this or just using an older version, but I definitely recommend bumping this up to 45s. Especial

Re: Kafka streams state store return hostname as unavailable when calling queryMetadataForKey method

2024-05-09 Thread Sophie Blee-Goldman
What version did you upgrade from? On Wed, May 8, 2024 at 10:32 PM Penumarthi Durga Prasad Chowdary < prasad.penumar...@gmail.com> wrote: > Hi Team, > I'm utilizing Kafka Streams to handle data from Kafka topics, running > multiple instances with the same application ID. This enables distribute

Re: Kafka streams state store return hostname as unavailable when calling queryMetadataForKey method

2024-05-13 Thread Sophie Blee-Goldman
ebalances/warmup tasks by the sawtooth pattern with an amplitude of 10 min, corresponding to the regular 10 minute probing rebalances. Hope this helps, Sophie On Thu, May 9, 2024 at 9:20 PM Penumarthi Durga Prasad Chowdary < prasad.penumar...@gmail.com> wrote: > Kafka upgraded from 3.5.1

Re: Explicitly creating topology topics in a streams app

2024-11-21 Thread Sophie Blee-Goldman
Just FYI Streams explicitly disables auto topic creation. This is because we want to detect eg accidental deletion of internal topics, since that can/will result in data loss. Better to shut down and get someone's attention so they can try and revive the deleted topic or decide what to do. Not ent