Currently, scala.Serdes.String, for example, invokes Serdes.String() once and caches the result.
However, the implementation of the String serde has a non-empty configure method that is variant in whether it's used as a key or value serde. So we won't get correct execution if we create one serde and use it for both keys and values.
Reviewers: Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
* Refactor the StreamThread main loop, in the following:
1. Fetch from consumer and enqueue data to tasks.
2. Check if any tasks should be enforced process.
3/ Loop over processable tasks and process them for N iterations, and then check for 1) commit, 2) punctuate, 3) need to call consumer.poll
4. Even if there is not data to process in this iteration, still need to check if commit / punctuate is needed
5. Finally, try update standby tasks.
*Add an optimization to only commit when it is needed (i.e. at least some process() or punctuate() was triggered since last commit).
*Found and fixed a ProducerFencedException scenario: producer.send() call would never throw a ProducerFencedException directly, but it may throw a KafkaException whose "cause" is a ProducerFencedException.
Reviewers: Matthias J. Sax <matthias@confluent.io>, John Roesler <john@confluent.io>, Bill Bejeck <bill@confluent.io>
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <guozhang@confluent.io>, Bill Bejeck <bill@confluent.io>, Kamal Chandraprakash <kamal.chandraprakash@gmail.com>
Previously, we depicted creating a Jackson serde for every pojo class, which becomes a burden in practice. There are many ways to avoid this and just have a single serde, so we've decided to model this design choice instead.
Reviewers: Viktor Somogyi <viktorsomogyi@gmail.com>, Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
This PR fixes the previously recursive call of Streams Scala peek
Reviewers: Joan Goyeau <joan@goyeau.com>, Guozhang Wang <guozhang@confluent.io>, John Roesler <john@confluent.io>
Changes:
1. Add org.apache.kafka.streams.processor.internals.metrics.CumulativeCount analogous to Count, but not a SampledStat
2. Use CumulativeCount for -total metrics in streams instead of Count
Testing strategy:
Add a test in StreamsMetricsImplTest which fails on old, incorrect behavior
The contribution is my original work and I license the work to the project under the project's open source license.
Reviewers: Guozhang Wang <guozhang@confluent.io>, John Roesler <john@confluent.io>
Update to KIP-328.
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <guozhang@confluent.io>, Ted Yu <yuzhihong@gmail.com>, Kamal Chandraprakash <kamal.chandraprakash@gmail.com>
Plus minor javadoc cleanups.
Reviewers: Matthias J. Sax <matthias@confluent.io>,Guozhang Wang <guozhang@confluent.io>, John Roesler <john@confluent.io>
Due to lack of conversion to kstream Predicate, existing filter method in KTable.scala would result in StackOverflowError.
This PR fixes the bug and adds testing for it.
Reviewers: Guozhang Wang <guozhang@confluent.io>, John Roesler <john@confluent.io>
Join in the Scala streams API is currently unusable in 2.0.0 as reported by @mowczare:
#5019 (comment)
This due to an overload of it with the same signature in the first curried parameter.
See compiler issue that didn't catch it: https://issues.scala-lang.org/browse/SI-2628
Reviewers: Debasish Ghosh <dghosh@acm.org>, Guozhang Wang <guozhang@confluent.io>, John Roesler <john@confluent.io>
This PR adds valueChangingOperation and mergeNode to StreamsGraphNode#toString
Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
Updated two integration tests to use IntegrationTestUtils#waitUntilFinalKeyValueRecordsReceived to eliminate flaky test results.
Also, I updated IntegrationTestUtils#waitUntilFinalKeyValueRecordsReceived method to support having results with the same key present with different values.
For testing, I ran the current suite of streams tests.
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
1) As titled, add a rewriteTopology that 1) sets application id, 2) maybe disable caching, 3) adjust for source KTable. This optimization can hence be applied for both DSL or PAPI generated Topology.
2) Defer the building of globalStateStores in rewriteTopology so that we can also disable caching. But we still need to build the state stores before InternalTopologyBuilder.build() since we should only build global stores once for all threads.
3) Added withCachingDisabled to StoreBuilder, it is a public API change.
4) [Optional] Fixed unit test config setting functionalities, and set the necessary config to shorten the unit test latency (now it reduces from 5min to 3.5min on my laptop).
Reviewers: Matthias J. Sax <matthias@confluent.io>, John Roesler <john@confluent.io>, Bill Bejeck <bill@confluent.io>, Ted Yu <yuzhihong@gmail.com>
This PR adds the optimization of eliminating multiple repartition topics when the KStream resulting from a key-changing operation executes other methods using the new key and reduces the repartition topics to one.
Note that this PR leaves in place the optimization for re-using a source topic as a changelog topic for source KTable instances. I'll have another follow-up PR to move the source topic optimization to a method within InternalStreamsBuilder so it can be performed in the same area of the code.
Additionally, the current value of StreamsConfig.OPTIMIZE is all and we'll need to have another KIP to change the value to 2.1.
An integration test RepartitionOptimizingIntegrationTest which asserts the same results for an optimized topology with one repartition topic as the un-optimized version with four repartition topics.
Reviewers: Matthias J. Sax <matthias@confluent.io>, John Roesler <john@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
Part I of KIP-238:
* add grace period to Windows
* deprecate retention/maintainMs and segmentInterval from Windows
* record expired records in the store with a new metric
* record late record drops as a new metric instead of as a "skipped record"
Reviewers: Matthias J. Sax <matthias@confluent.io>, Bill Bejeck <bill@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
#5468 introduced a breaking API change that was actually avoidable. This PR re-introduces the old API as deprecated and alters the API introduced by #5468 to be consistent with the other methods
also, fixed misc syntax problems
- fix log statement in Topology Builder.
- addressed some warnings shown by Intellij
Reviewers: Viktor Somogyi <viktorsomogyi@gmail.com>, Satish Duggana <satishd@apache.org>, Matthias J. Sax <matthias@confluent.io>
While working on 4th PR, I noticed that I had missed adding stores via the graph vs. directly via the InternalStreamsBuilder. Probably ok to do so, but we should be consistent.
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
While debugging the reported issue, I found that our current unit test lacks coverage to actually expose the underlying root cause.
Reviewers: Bill Bejeck <bill@confluent.io>, Matthias J. Sax <matthias@confluent.io>
1. In each iteration, decide if a task is processable if all of its partitions contains data, so it can decide which record to process next.
1.a Add one exception that, if the task indeed have data on some but not all of its partitions, we only consider as not processable for some finite round of iterations.
1.b Add a task-level metric to record whenever we are forced to process a task that is only "partially data available", since it may leads to non-determinism.
2. Break the main loop on put-raw-data and process-them. Since now not all data put into the queue would be processed completely within a single iteration.
3. NOTE that within an iteration, if a task has exhausted one of its queue it will still be processed, since we only update processable list once in each iteration, I'm improving on this on the follow-up part III PR.
4. Found and fixed a bug in metrics recording: the taskName and sensorName parameters were exchanged.
5. Optimized task stream time computation again since our current partition stream time reasoning has been simplified.
6. Added unit tests.
Reviewers: Matthias J. Sax <matthias@confluent.io>, John Roesler <vvcephei@users.noreply.github.com>, Bill Bejeck <bbejeck@gmail.com>
The specific changes in this PR from the second PR include:
1. Changed the types of graph nodes to names conveying more context
2. Build the entire physical plan from the graph, after StreamsBuilder.build() is called.
Other changes are addressed directly as review comments on the PR.
Testing consists of using all existing streams tests to validate building the physical plan with graph
Reviewers: Matthias J. Sax <matthias@confluent.io>, John Roesler <vvcephei@users.noreply.github.com>, Guozhang Wang <wangguoz@gmail.com>
Use delivery timeout instead of retries when possible and remove various TODOs associated with completion of KIP-91.
Reviewers: Ismael Juma <ismael@juma.me.uk>, Guozhang Wang <wangguoz@gmail.com>
* new minimum is 0, just like window size
* refactor tests to use smaller segment sizes as well
Reviewers: Matthias J. Sax <matthias@confluent.io>, Guozhang Wang <wangguoz@gmail.com>
1. When we reinitialize the state store due to no CHECKPOINT with EOS turned on, we should update the checkpoint to consumer.seekToBeginnning() / consumer.position() to avoid falling into endless iterations.
2. Fixed a few other logic bugs around needsInitializing and needsRestoring.
Reviewers: Jason Gustafson <jason@confluent.io>, Bill Bejeck <bbejeck@gmail.com>