Jon,
perhaps you could share the full integration test (or whatever code you're
using to experiment)? We had a similar "how does X work?" question on
StackOverflow recently [1], and it was much easier to help once we e.g.
also understood how the test data was exactly being generated.
-Michael
I remain more than mystified about the workings of the StateStore. I tried
making aggregations with a 1minute window, 10 second advance and a _12
hour_ retention (which is longer than the retention.ms of the topic). I
still couldn't get more than a 15% hit rate on the StateStore.
Are there config
It's based in "stream time", ie, the internally tracked progress based
on the timestamps return by TimestampExtractor.
-Matthias
On 3/29/17 12:52 PM, Jon Yeargers wrote:
> So '.until()' is based on clock time / elapsed time (IE record age) /
> something else?
>
> The fact that Im seeing lots of
So '.until()' is based on clock time / elapsed time (IE record age) /
something else?
The fact that Im seeing lots of records come through that can't be found in
the Store - these are 'old' and already expired?
Going forward - it would be useful to have different forms of '.until()' so
one could
Jon,
You should be able to query anything that has not expired, i.e., based on
TimeWindows.until(..).
Thanks,
Damian
On Wed, 29 Mar 2017 at 17:24 Jon Yeargers wrote:
> To be a bit more specific:
>
> If I call this: KTable kt =
> sourceStream.groupByKey().reduce(..., "somekeystore");
>
To be a bit more specific:
If I call this: KTable kt =
sourceStream.groupByKey().reduce(..., "somekeystore");
and then call this:
kt.forEach()-> ...
Can I assume that everything that comes out will be available in
"somekeystore"? If not, what subset should I expect to find there?
On We
But if a key shows up in a KTable->forEach should it be available in the
StateStore (from the KTable)?
On Wed, Mar 29, 2017 at 6:31 AM, Michael Noll wrote:
> Jon,
>
> there's a related example, using a window store and a key-value store, at
> https://github.com/confluentinc/examples/blob/3.
> 2.
Jon,
there's a related example, using a window store and a key-value store, at
https://github.com/confluentinc/examples/blob/3.2.x/kafka-streams/src/test/java/io/confluent/examples/streams/ValidateStateWithInteractiveQueriesLambdaIntegrationTest.java
(this is for Confluent 3.2 / Kafka 0.10.2).
-M
Im only running one instance (locally) to keep things simple.
Reduction:
KTable, String> hourAggStore =
sourceStream.groupByKey().reduce(rowReducer,
TimeWindows.of(65 * 60 * 1000L).advanceBy(5 * 60 *
1000).until(70 * 60 * 1000L),
"HourAggStore");
then I ge
Hi Jon,
If you are able to get a handle on the store, i.e., via
KafkaStreams.store(...) and call fetch without any exceptions, then the
store is available.
The time params to fetch are the boundaries to search for windows for the
given key. They relate to the start time of the window, so if you di
10 matches
Mail list logo