ote:
>
> > Thanks for the KIP.
> > +1 (binding)
> >
> > -Bill
> >
> >
> > On Wed, Nov 6, 2019 at 1:22 AM Matthias J. Sax
> > wrote:
> >
> > > +1 (binding)
> > >
> > >
> > > On 10/31/19 10:52 AM, Walker Carls
/19 10:52 AM, Walker Carlson wrote:
> > > Hello all,
> > >
> > > I'd like to call a vote on the updated KIP-150: Kafka-Streams Cogroup
> > > found here
> > > <
> >
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup
> > >
> > >
> > > Thanks,
> > > Walker
> > >
> >
> >
>
--
-- Guozhang
Thanks for the KIP.
+1 (binding)
-Bill
On Wed, Nov 6, 2019 at 1:22 AM Matthias J. Sax
wrote:
> +1 (binding)
>
>
> On 10/31/19 10:52 AM, Walker Carlson wrote:
> > Hello all,
> >
> > I'd like to call a vote on the updated KIP-150: Kafka-Streams C
+1 (binding)
On 10/31/19 10:52 AM, Walker Carlson wrote:
> Hello all,
>
> I'd like to call a vote on the updated KIP-150: Kafka-Streams Cogroup
> found here
> <https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup>
>
> Than
>>>>>>>>>>
> > >>>>>>>>>>
> > >>>>>>>>>>
> > >>>>>>>>>> On Wed, Oct 23, 2019 at 9:58 PM Sophie Blee-Goldman <
> > >>>>>>>> sop...@confluent.io
> > >>>>>>>>>&g
Hello all,
I'd like to call a vote on the updated KIP-150: Kafka-Streams Cogroup
found here
<https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup>
Thanks,
Walker
>>>>>>>>>>>
> >>>>>>>>>>> One last question I have then is about the
> >>>>>>> operator/store/repartition
> >>>>>>>>>>> naming -- seems like
> >>>>>>>>>>>
confluent.io>
>>>>>>>>>>> wrote:
>>>>>>>>>>>
>>>>>>>>>>>> Interesting idea, Sophie.
>>>>>>>>>>>>
>>>>>>>>>>>> So f
; > > now
> > > > > > > (or
> > > > > > > > >>>> maybe the
> > > > > > > > >>>> answer seems obviously "no") but we seem to often end up
> > > needing
> > > > > > to
> > > > > > > > add
> > > > > > > > >>> new
> > > > > > > > >>>> overloads and/or deprecate old ones as new features or
>
t; > > > > object,
> > > > > > > >>>> which itself would have an initializer and a materialized.
> > If we
> > > > > > ever
> > > > > > > >>> need
> > > &
Oct 23, 2019 at 10:49 AM Walker Carlson <
> > > > > > wcarl...@confluent.io
> > > > > > >>>
> > > > > > >>>> wrote:
> > > > > > >>>>
> > > > > > >>>>> Hi Sophie,
> &g
gt; > > > > >>>>> first made the method is called on a groupedStream already.
> > > However
> > > > > >> each
> > > > > >>>>> subsequent stream-aggregator pair is added on to a cogroup
> > stream
> > >
t; > >>>>> can collect many grouped streams with overlapping key spaces
> and
> > > any
> > > > >>> kind
> > > > >>>>> of value types. Once aggregated its value will be reduced into
> > one
> > > > >> type.
; that
> > > > >>> you
> > > > >>>>> can collect many grouped streams with overlapping key spaces
> and
> > > any
> > > > >>> kind
> > > > >>>>> of value types. Once aggregated its value will be reduced into
> > one
> > > > >> type
; > >>>>>
> > > >>>>> Thanks,
> > > >>>>> Walker
> > > >>>>>
> > > >>>>> On Tue, Oct 22, 2019 at 8:59 PM Sophie Blee-Goldman <
> > > >>&
oes that make sense?
> > > >>>>>
> > > >>>>> This is a good question and I will include this explanation in
> the
> > > kip
> > > >>> as
> > > >>>>> well.
> > > >>>>>
> > > >>>>> Than
gt;
> > >>>>>> 1) It seems a little awkward to me that with the current API, we
> > >> have a
> > >>>>>> nearly identical
> > >>>>>> "add stream to cogroup" method, except for the first which has a
> > >>>>> different
>
;>>>>> are joined as .cogroup(Stream, Aggregator) ). I'm not sure what it
> >>> would
> >>>>>> look like exactly,
> >>>>>> but I was just wondering if you'd considered and/or rejected any
> >>>>>> alternative A
gt; >>> would
> >>>>>> look like exactly,
> >>>>>> but I was just wondering if you'd considered and/or rejected any
> >>>>>> alternative APIs?
> >>>>>>
> >>>>>> 2) This might just be my lack of familiarity
he user seems to have some control over
>> how
>>>>>> exactly
>>>>>> the different streams are joined through the ValueJoiners. Would this
>>> new
>>>>>> cogroup
>>>>>> simply concatenate the values from the differe
those of us
> who
> > >>> aren't fluent
> > >>> in cogroup semantics :)
> > >>>
> > >>> Cheers,
> > >>> Sophie
> > >>>
> > >>> On Thu, Oct 17, 2019 at 3:06 PM Walker Carlson <
> wcarl...
;> aren't fluent
> >>> in cogroup semantics :)
> >>>
> >>> Cheers,
> >>> Sophie
> >>>
> >>> On Thu, Oct 17, 2019 at 3:06 PM Walker Carlson
> >>> wrote:
> >>>
> >>>> Good catch I update
gt;>
>>>> I have made a PR for this KIP
>>>>
>>>> I then am splitting it into 3 parts, first cogroup for a key-value
>> store
>>> (
>>>> here <https://github.com/apache/kafka/pull/7538>), then for a
>>>> timeWindow
confluent.io>
> > > wrote:
> > >
> > > > Walker,
> > > >
> > > > thanks for picking up the KIP and reworking it for the changed API.
> > > >
> > > > Overall, the updated API suggestions make sense to me.
artitioning.
> > >
> > > Walker
> > >
> > > On Tue, Oct 15, 2019 at 12:47 PM Matthias J. Sax
> > > wrote:
> > >
> > > > Walker,
> > > >
> > > > thanks for picking up the KIP and reworking it for the changed API.
or the changed API.
> > >
> > > Overall, the updated API suggestions make sense to me. The seem to
> align
> > > quite nicely with our current API design.
> > >
> > > One nit: In `CogroupedKStream#aggregate(...)` the type parameter of
> > &g
e to me. The seem to align
> > quite nicely with our current API design.
> >
> > One nit: In `CogroupedKStream#aggregate(...)` the type parameter of
> > `Materialized` should be `V`, not `VR`?
> >
> >
> > -Matthias
> >
> >
> >
> > On
pedKStream#aggregate(...)` the type parameter of
> `Materialized` should be `V`, not `VR`?
>
>
> -Matthias
>
>
>
> On 10/14/19 2:57 PM, Walker Carlson wrote:
> >
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup
> > here
`VR`?
-Matthias
On 10/14/19 2:57 PM, Walker Carlson wrote:
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup
> here
> is a link
>
> On Mon, Oct 14, 2019 at 2:52 PM Walker Carlson
> wrote:
>
>> Hello all,
>>
>> I hav
https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup
here
is a link
On Mon, Oct 14, 2019 at 2:52 PM Walker Carlson
wrote:
> Hello all,
>
> I have picked up and updated KIP-150. Due to changes to the project since
> KIP #150 was written there are a fe
Hello all,
I have picked up and updated KIP-150. Due to changes to the project since
KIP #150 was written there are a few items that need to be updated.
First item that changed is the adoption of the Materialized parameter.
The second item is the WindowedBy. How the old KIP handles windowing is
stomer[
> > > > > > > >> >
> > > > > > > >> > cart:{Item[no:01]},
> > > > > > > >>
hases:{},
> > > > > > >> > wishList:{}
> > > > > > >> > ]
> > > > > > >> >
> > > > > > >> >
The KIP and PR have been updated please go take a look and vote.
For those worried about the [DISCUSS] Streams DSL/StateStore Refactoring
email thread affecting this I believe the cogroup methods fit well into the
streams dsl and won't need to change. We can update the aggregate methods
in the sam
+1
Thanks,
Bill
On Wed, Jun 14, 2017 at 8:10 PM, Xavier Léauté wrote:
> +1 from me
>
> any stream should be able to initialize the cogroup
>
> On Wed, Jun 14, 2017 at 3:44 PM Kyle Winkelman
> wrote:
>
> > I will update the kip to have only the aggregator in the first cogroup
> call
> > and the
+1 from me
any stream should be able to initialize the cogroup
On Wed, Jun 14, 2017 at 3:44 PM Kyle Winkelman
wrote:
> I will update the kip to have only the aggregator in the first cogroup call
> and the initializer and serde in the aggregate calls.
>
> This seems to be the consensus on the em
I will update the kip to have only the aggregator in the first cogroup call
and the initializer and serde in the aggregate calls.
This seems to be the consensus on the email chain.
Thanks,
Kyle
On Jun 14, 2017 5:41 PM, wrote:
That is not the case. No matter which stream the record comes in on t
That is not the case. No matter which stream the record comes in on the
initializer is called if there is not currently an object in the store.
On Jun 14, 2017 5:12 PM, "Guozhang Wang" wrote:
> While regarding where we should ask users to set serdes: I think I'm
> convinced by Xavier that they s
While regarding where we should ask users to set serdes: I think I'm
convinced by Xavier that they should be in the `aggregate` call (but only
those does not pass in a state store supplier) instead of the
`KStream#cogroup` call to be consistent with other aggregate functions.
BTW another motivatio
To clarify it isn't required to have the initializer in the first cogroup
because the first aggregator will have the value type. I like how the
initializer makes it abundantly clear that the final type will be that.
Right now I'm split because the case may be made that you want to supply a
differen
I'd suggest we do not block this KIP until the serde work has been sorted
out: we cannot estimate yet how long it will take yet. Instead let's say
make an agreement on where we want to specify the serdes: whether on the
first co-group call or on the aggregate call.
Also about the initializer speci
+1 on deferring discussion on Serdes until API improvements are ironed out.
On Tue, Jun 13, 2017 at 2:06 PM, Matthias J. Sax
wrote:
> Hi,
>
> I am just catching up on this thread. (1) as most people agree, we
> should not add anything to KStreamBuilder (btw: we actually plan to move
> #merge() t
Hi,
I am just catching up on this thread. (1) as most people agree, we
should not add anything to KStreamBuilder (btw: we actually plan to move
#merge() to KStream and deprecate it on KStreamBuilder as it's a quite
unnatural API atm).
About specifying Serdes: there is still the idea to improve to
You're right, I haven't thought of that.
Cheers,
Michał
On 13/06/17 13:00, Kyle Winkelman wrote:
First, I would prefer not calling it aggregate because there are already
plenty of aggregate methods.
Second, I dont think this would really work because after each aggregate
you now have a uniqu
First, I would prefer not calling it aggregate because there are already
plenty of aggregate methods.
Second, I dont think this would really work because after each aggregate
you now have a unique KTable (someone may want a table with 4 streams and
reuse those 4 in another table but with one more
I think we are discussing two separate things here, so it might be worth
clarifying:
1) the position of the initializer with respect to the aggregators. If I
understand correctly, Guozhang seems to think it is more natural to specify
the initializer first, despite it not bearing any relation to th
> > > > >> >
> > > > > >> > cart:{Item[no:01]},
> > > > > >> > purchases:{Item[no:07],Item[no:08]},
> > > > > >&
> ]
> > > > >> >
> > > > >> > 1L, Customer[
> > > > >> >
> > > > >> > cart:{Item[no:01]},
> > > > >> >
at
> > >> > > >>>>> Jay’s
> > >> > > >>>>>>>>> example
> > >> > > >>>>>>>>>>> is,
> > >> > > >>>>>>>>>>>>>
the KIP again. A couple of comments:
> > >> > > >>>>>>>>>>>>>
> > >> > > >>>>>>>>>>>>> - minor: could you add an exact example (similar to
> > what
> > >> > > >>>>> Jay’s
> > >> > > &
; >> > > >>>>>>>>>>> In
> >> > > >>>>>>>>>>>>> an ideal world, an optimizer would take the existing
> DSL
> >> > > >>>>> and do
> >> > > >>>>>>>> the
> >> &g
wishList:{}
> > > >> > ]
> > > >> >
> > > >> > 1L, Customer[
> > > >> >
> > > >> > cart:{Item[no:01]},
> > > >> > purchases:{Item[no:07],Item[no
gt; > >>>>>>>>>>> your
>> > > >>>>>>>>>>>>> thoughts on whether it’s possible to do this
>> > > >> optimization
>> > > >>>>> with
>> > >
gt;>>>>>> Thanks
> > > >>>>>>>>>>>>> Eno
> > > >>>>>>>>>>>>>
> > > >>>>>>>>>>>>>> On May 5, 2017, at 4:39 PM, Jay Kreps <
> > > >> j...@confluent.io>
> &
; > >>>>>>>>>>>>> case
> > >>>>>>>>>>>>>> we often use. In that use case you have a dozen systems
> > >>>>> each
> > >>>>>> of
> > >>>>>>>>
> >> >
> > >> > ...
> > >> >
> > >> >
> > >> > I'm wondering if it makes more sense to only start sending the
> update
> > if
> > >> > the corresponding agg-key has seen at least one input from each of
> the
> > >> > input stream? Maybe it is out of the scope of this KIP and we can
> make
> > >> it a
> > >> > more general discussion in a separate one.
> > >> >
> > >> >
> > >> > Guozhang
> > >> >
> > >> >
> > >> > On Fri, May 19, 2017 at 8:37 AM, Xavier Léauté >
> > >> > wrote:
> > >> >
> > >> > > Hi Kyle, I left a few more comments in the discussion thread, if
> you
> > >> > > wouldn't mind taking a look
> > >> > >
> > >> > > On Fri, May 19, 2017 at 5:31 AM Kyle Winkelman <
> > >> winkelman.k...@gmail.com
> > >> > >
> > >> > > wrote:
> > >> > >
> > >> > > > Hello all,
> > >> > > >
> > >> > > > I would like to start the vote on KIP-150.
> > >> > > >
> > >> > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+
> > >> > > Kafka-Streams+Cogroup
> > >> > > >
> > >> > > > Thanks,
> > >> > > > Kyle
> > >> > > >
> > >> > >
> > >> >
> > >> >
> > >> >
> > >> > --
> > >> > -- Guozhang
> > >> >
> > >>
> > >
> >
>
t;>>>>>>>>>> join/munge these into a single profile record for each
> >>>>>> customer
> >>>>>>>>> that
> >>>>>>>>>>> has
> >>>>>>>>>>>>>> all the relevant i
ngle stage to group all these things
>>> that
>>>>>> are
>>>>>>>>>> already
>>>>>>>>>>>>>> co-partitioned. A distributed database would do this
>>> under
>>>>> the
>>>
> like
> > >> > >> > > > >>>> it
> > >> > >> > > > >>>>> could be a useful optimization.
> > >> > >> > > > >>>>>
> > >> > >> > > > >>
t; >> > > > >>>>>> and make more of a step by step description:
> >> > >> > > > >>>>>>
> >> > >> > > > >>>>>> Example with Current API:
> >> > >> > > > >>>>>>
> >
> initializer3
>> > >> ,
>> > >> > > > >>>>> aggregator3,
>> > >> > > > >>>>>> aggValueSerde3, storeName3);
>> > >> > > > >>>>>> KTable cogr
t; > >>>>>> joiners are more like mergers, or second make them
> > >> intermediate
> > >> > > > states
> > >> > > > >>>>> such
> > >> > > > >>>>>> as Topic1Map, Topic2Map, a
storeName1.
> >> > > It
> >> > > > >>>> will
> >> > > > >>>>>> produce this in the form of the first intermediate value
> and
> >> get
> >> > > > sent
> >> > > > >>>>>> through a KTabl
>
> >> > I'm wondering if it makes more sense to only start sending the update
> if
> >> > the corresponding agg-key has seen at least one input from each of the
> >> > input stream? Maybe it is out of the scope of this KIP and we can make
> >> it a
> >> > more general discussion in a separate one.
> >> >
> >> >
> >> > Guozhang
> >> >
> >> >
> >> > On Fri, May 19, 2017 at 8:37 AM, Xavier Léauté
> >> > wrote:
> >> >
> >> > > Hi Kyle, I left a few more comments in the discussion thread, if you
> >> > > wouldn't mind taking a look
> >> > >
> >> > > On Fri, May 19, 2017 at 5:31 AM Kyle Winkelman <
> >> winkelman.k...@gmail.com
> >> > >
> >> > > wrote:
> >> > >
> >> > > > Hello all,
> >> > > >
> >> > > > I would like to start the vote on KIP-150.
> >> > > >
> >> > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+
> >> > > Kafka-Streams+Cogroup
> >> > > >
> >> > > > Thanks,
> >> > > > Kyle
> >> > > >
> >> > >
> >> >
> >> >
> >> >
> >> > --
> >> > -- Guozhang
> >> >
> >>
> >
>
>>>>>>
>> > > > >>>>>> If you think through all possibilities for incoming topics
>> you
>> > > will
>> > > > >> see
>> > > > >>>>>> that no matter which top
>> > input stream? Maybe it is out of the scope of this KIP and we can make
>> it a
>> > more general discussion in a separate one.
>> >
>> >
>> > Guozhang
>> >
>> >
>> > On Fri, May 19, 2017 at 8:37 AM, Xavier Léauté
>> > wrote:
>> >
>> > > Hi Kyle, I left a few more comments in the discussion thread, if you
>> > > wouldn't mind taking a look
>> > >
>> > > On Fri, May 19, 2017 at 5:31 AM Kyle Winkelman <
>> winkelman.k...@gmail.com
>> > >
>> > > wrote:
>> > >
>> > > > Hello all,
>> > > >
>> > > > I would like to start the vote on KIP-150.
>> > > >
>> > > > https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+
>> > > Kafka-Streams+Cogroup
>> > > >
>> > > > Thanks,
>> > > > Kyle
>> > > >
>> > >
>> >
>> >
>> >
>> > --
>> > -- Guozhang
>> >
>>
>
f the scope of this KIP and we can make
> it a
> > more general discussion in a separate one.
> >
> >
> > Guozhang
> >
> >
> > On Fri, May 19, 2017 at 8:37 AM, Xavier Léauté
> > wrote:
> >
> > > Hi Kyle, I left a few more comments in
on in a separate one.
>
>
> Guozhang
>
>
> On Fri, May 19, 2017 at 8:37 AM, Xavier Léauté
> wrote:
>
> > Hi Kyle, I left a few more comments in the discussion thread, if you
> > wouldn't mind taking a look
> >
> > On Fri, May 19, 2017 at 5:31 AM Ky
gt;> If you think through all possibilities for incoming topics
>> you
>> > > will
>> > > > >> see
>> > > > >>>>>> that no matter which topic it comes in through all three
>> stores
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+
> Kafka-Streams+Cogroup
> >
> > Thanks,
> > Kyle
> >
>
--
-- Guozhang
h Proposed API:
> > > > >>>>>>
> > > > >>>>>> KGroupedStream grouped1 = builder.stream("topic1").
> > > > >>>> groupByKey();
> > > > >>>>>> KGroupedStream group
t;>>
> > > >>>>>> As you can see this creates 1 StateStore, requires 1
> initializer,
> > > and
> > > >> 1
> > > >>>>>> aggValueSerde. The user no longer has to worry about the
> > > intermediate
> >
Hi Kyle, I left a few more comments in the discussion thread, if you
wouldn't mind taking a look
On Fri, May 19, 2017 at 5:31 AM Kyle Winkelman
wrote:
> Hello all,
>
> I would like to start the vote on KIP-150.
>
> https://cwiki.apache.org/confluence/display/KAFKA/KIP
>>> KStreamAggreagte and grab the current aggregate from storeName1.
> It
> > >>>> will
> > >>>>>> add its incoming object to the aggregate, update the store and
> pass
> > >> the
> > >>>>>
Hello all,
I would like to start the vote on KIP-150.
https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup
Thanks,
Kyle
build
>>>>>>>>>>>>> the final aggregate CG value. This is something the user
>> could
>>>>>>> avoid
>>>>>>>>>>>>> thinking about with this KIP.
>>>>>>>>>>
e it will look up the current value
> > of
> > > >>> the
> > > >>>>>>> key
> > > >>>>>>>> in
> > > >>>>>>>>> storeName3 and use the second joiner to build
three
stores
> > are
> > >>>>>>>> queried
> > >>>>>>>>> and all of the joiners must get used.
> > >>>>>>>>>
> > >>>>>>>>> Topology wise for N incoming streams this creates
it comes in through all three stores
> > are
> > >>>>>>>> queried
> > >>>>>>>>> and all of the joiners must get used.
> > >>>>>>>>>
> > >>>>>>>>&
gt; >>>>>>>>> KGroupedStream grouped2 = builder.stream("topic2").
> >>>>>>> groupByKey();
> >>>>>>>>> KGroupedStream grouped3 = builder.stream("topic3").
> >>>>>>> groupByKey();
> >>>>>>>>> KTable cogrouped = grouped1.cogroup(initializer1,
> >>> aggregator1,
> >>>>>>>>> aggValueSerde1, storeName1)
> >>>>>>>>> .cogroup(grouped2, aggregator2)
> >>>>>>>>> .cogroup(grouped3, aggregator3)
> >>>>>>>>> .aggregate();
> >>>>>>>>>
> >>>>>>>>> As you can see this creates 1 StateStore, requires 1 initializer,
> >>> and
> >>>>> 1
> >>>>>>>>> aggValueSerde. The user no longer has to worry about the
> >>> intermediate
> >>>>>>>>> values and the joiners. All they have to think about is how each
> >>>>> stream
> >>>>>>>>> impacts the creation of the final CG object.
> >>>>>>>>>
> >>>>>>>>> When a new input arrives lets say at "topic1" it will first go
> >>> through
> >>>>>>> a
> >>>>>>>>> KStreamAggreagte and grab the current aggregate from storeName1.
> It
> >>>>>>> will
> >>>>>>>>> add its incoming object to the aggregate, update the store and
> pass
> >>>>> the
> >>>>>>>> new
> >>>>>>>>> aggregate on. This new aggregate goes through the KStreamCogroup
> >>> which
> >>>>>>> is
> >>>>>>>>> pretty much just a pass through processor and you are done.
> >>>>>>>>>
> >>>>>>>>> Topology wise for N incoming streams the new api will only every
> >>>>>>> create N
> >>>>>>>>> KStreamAggregates and 1 KStreamCogroup.
> >>>>>>>>>
> >>>>>>>>> On Thu, May 4, 2017 at 4:42 PM, Matthias J. Sax <
> >>>>> matth...@confluent.io
> >>>>>>>>
> >>>>>>>>> wrote:
> >>>>>>>>>
> >>>>>>>>>> Kyle,
> >>>>>>>>>>
> >>>>>>>>>> thanks a lot for the KIP. Maybe I am a little slow, but I could
> >>> not
> >>>>>>>>>> follow completely. Could you maybe add a more concrete example,
> >>> like
> >>>>>>> 3
> >>>>>>>>>> streams with 3 records each (plus expected result), and show the
> >>>>>>>>>> difference between current way to to implement it and the
> proposed
> >>>>>>> API?
> >>>>>>>>>> This could also cover the internal processing to see what store
> >>> calls
> >>>>>>>>>> would be required for both approaches etc.
> >>>>>>>>>>
> >>>>>>>>>> I think, it's pretty advanced stuff you propose, and it would
> >>> help to
> >>>>>>>>>> understand it better.
> >>>>>>>>>>
> >>>>>>>>>> Thanks a lot!
> >>>>>>>>>>
> >>>>>>>>>>
> >>>>>>>>>> -Matthias
> >>>>>>>>>>
> >>>>>>>>>>
> >>>>>>>>>>
> >>>>>>>>>> On 5/4/17 11:39 AM, Kyle Winkelman wrote:
> >>>>>>>>>>> I have made a pull request. It can be found here.
> >>>>>>>>>>>
> >>>>>>>>>>> https://github.com/apache/kafka/pull/2975
> >>>>>>>>>>>
> >>>>>>>>>>> I plan to write some more unit tests for my classes and get
> >>> around
> >>>>>>> to
> >>>>>>>>>>> writing documentation for the public api additions.
> >>>>>>>>>>>
> >>>>>>>>>>> One thing I was curious about is during the
> >>>>>>>>>> KCogroupedStreamImpl#aggregate
> >>>>>>>>>>> method I pass null to the KGroupedStream#repartitionIfRequired
> >>>>>>>> method.
> >>>>>>>>> I
> >>>>>>>>>>> can't supply the store name because if more than one grouped
> >>> stream
> >>>>>>>>>>> repartitions an error is thrown. Is there some name that
> someone
> >>>>>>> can
> >>>>>>>>>>> recommend or should I leave the null and allow it to fall back
> to
> >>>>>>> the
> >>>>>>>>>>> KGroupedStream.name?
> >>>>>>>>>>>
> >>>>>>>>>>> Should this be expanded to handle grouped tables? This would be
> >>>>>>>> pretty
> >>>>>>>>>> easy
> >>>>>>>>>>> for a normal aggregate but one allowing session stores and
> >>> windowed
> >>>>>>>>>> stores
> >>>>>>>>>>> would required KTableSessionWindowAggregate and
> >>>>>>> KTableWindowAggregate
> >>>>>>>>>>> implementations.
> >>>>>>>>>>>
> >>>>>>>>>>> Thanks,
> >>>>>>>>>>> Kyle
> >>>>>>>>>>>
> >>>>>>>>>>> On May 4, 2017 1:24 PM, "Eno Thereska" >
> >>>>>>>> wrote:
> >>>>>>>>>>>
> >>>>>>>>>>>> I’ll look as well asap, sorry, been swamped.
> >>>>>>>>>>>>
> >>>>>>>>>>>> Eno
> >>>>>>>>>>>>> On May 4, 2017, at 6:17 PM, Damian Guy >
> >>>>>>>> wrote:
> >>>>>>>>>>>>>
> >>>>>>>>>>>>> Hi Kyle,
> >>>>>>>>>>>>>
> >>>>>>>>>>>>> Thanks for the KIP. I apologize that i haven't had the chance
> >>> to
> >>>>>>>> look
> >>>>>>>>>> at
> >>>>>>>>>>>>> the KIP yet, but will schedule some time to look into it
> >>>>>>> tomorrow.
> >>>>>>>>> For
> >>>>>>>>>>>> the
> >>>>>>>>>>>>> implementation, can you raise a PR against kafka trunk and
> mark
> >>>>>>> it
> >>>>>>>> as
> >>>>>>>>>>>> WIP?
> >>>>>>>>>>>>> It will be easier to review what you have done.
> >>>>>>>>>>>>>
> >>>>>>>>>>>>> Thanks,
> >>>>>>>>>>>>> Damian
> >>>>>>>>>>>>>
> >>>>>>>>>>>>> On Thu, 4 May 2017 at 11:50 Kyle Winkelman <
> >>>>>>>> winkelman.k...@gmail.com
> >>>>>>>>>>
> >>>>>>>>>>>> wrote:
> >>>>>>>>>>>>>
> >>>>>>>>>>>>>> I am replying to this in hopes it will draw some attention
> to
> >>> my
> >>>>>>>> KIP
> >>>>>>>>>> as
> >>>>>>>>>>>> I
> >>>>>>>>>>>>>> haven't heard from anyone in a couple days. This is my first
> >>> KIP
> >>>>>>>> and
> >>>>>>>>>> my
> >>>>>>>>>>>>>> first large contribution to the project so I'm sure I did
> >>>>>>>> something
> >>>>>>>>>>>> wrong.
> >>>>>>>>>>>>>> ;)
> >>>>>>>>>>>>>>
> >>>>>>>>>>>>>> On May 1, 2017 4:18 PM, "Kyle Winkelman" <
> >>>>>>>> winkelman.k...@gmail.com>
> >>>>>>>>>>>> wrote:
> >>>>>>>>>>>>>>
> >>>>>>>>>>>>>>> Hello all,
> >>>>>>>>>>>>>>>
> >>>>>>>>>>>>>>> I have created KIP-150 to facilitate discussion about
> adding
> >>>>>>>>> cogroup
> >>>>>>>>>> to
> >>>>>>>>>>>>>>> the streams DSL.
> >>>>>>>>>>>>>>>
> >>>>>>>>>>>>>>> Please find the KIP here:
> >>>>>>>>>>>>>>> https://cwiki.apache.org/confluence/display/KAFKA/KIP-
> >>>>>>>>>>>>>>> 150+-+Kafka-Streams+Cogroup
> >>>>>>>>>>>>>>>
> >>>>>>>>>>>>>>> Please find my initial implementation here:
> >>>>>>>>>>>>>>> https://github.com/KyleWinkelman/kafka
> >>>>>>>>>>>>>>>
> >>>>>>>>>>>>>>> Thanks,
> >>>>>>>>>>>>>>> Kyle Winkelman
> >>>>>>>>>>>>>>>
> >>>>>>>>>>>>>>
> >>>>>>>>>>>>
> >>>>>>>>>>>>
> >>>>>>>>>>>
> >>>>>>>>>>
> >>>>>>>>>>
> >>>>>>>>>
> >>>>>>>>
> >>>>>>>
> >>>>>
> >>>>>
> >>>
> >>>
>
>
group(grouped2, aggregator2)
>>>>>>>>> .cogroup(grouped3, aggregator3)
>>>>>>>>> .aggregate();
>>>>>>>>>
>>>>>>>>> As you can see this creates 1 StateStore, requires 1 initializer,
>>
at "topic1" it will first go
>> through
>> >>>> a
>> >>>>>> KStreamAggreagte and grab the current aggregate from storeName1. It
>> >>>> will
>> >>>>>> add its incoming object to the aggregate, update the s
<
> >> matth...@confluent.io
> >>>>>
> >>>>>> wrote:
> >>>>>>
> >>>>>>> Kyle,
> >>>>>>>
> >>>>>>> thanks a lot for the KIP. Maybe I am a little
tween current way to to implement it and the proposed
>>>> API?
>>>>>>> This could also cover the internal processing to see what store calls
>>>>>>> would be required for both approaches etc.
>>>>>>>
>>>>>>&
I have made a pull request. It can be found here.
> >>>>>>
> >>>>>> https://github.com/apache/kafka/pull/2975
> >>>>>>
> >>>>>> I plan to write some more unit tests for my classes and get around
> >> to
>
urious about is during the
>>>>> KCogroupedStreamImpl#aggregate
>>>>>> method I pass null to the KGroupedStream#repartitionIfRequired
>>> method.
>>>> I
>>>>>> can't supply the store name because if more than one grouped stream
>>&
would required KTableSessionWindowAggregate and
> KTableWindowAggregate
> > > > > implementations.
> > > > >
> > > > > Thanks,
> > > > > Kyle
> > > > >
> > > > > On May 4, 2017 1:24 PM, "Eno Th
PM, Damian Guy
> wrote:
> > > >>>
> > > >>> Hi Kyle,
> > > >>>
> > > >>> Thanks for the KIP. I apologize that i haven't had the chance to
> look
> > > at
> > > >>> the KIP yet, but will schedule some time to look into it tomorrow.
> > For
> > > >> the
> > > >>> implementation, can you raise a PR against kafka trunk and mark it
> as
> > > >> WIP?
> > > >>> It will be easier to review what you have done.
> > > >>>
> > > >>> Thanks,
> > > >>> Damian
> > > >>>
> > > >>> On Thu, 4 May 2017 at 11:50 Kyle Winkelman <
> winkelman.k...@gmail.com
> > >
> > > >> wrote:
> > > >>>
> > > >>>> I am replying to this in hopes it will draw some attention to my
> KIP
> > > as
> > > >> I
> > > >>>> haven't heard from anyone in a couple days. This is my first KIP
> and
> > > my
> > > >>>> first large contribution to the project so I'm sure I did
> something
> > > >> wrong.
> > > >>>> ;)
> > > >>>>
> > > >>>> On May 1, 2017 4:18 PM, "Kyle Winkelman" <
> winkelman.k...@gmail.com>
> > > >> wrote:
> > > >>>>
> > > >>>>> Hello all,
> > > >>>>>
> > > >>>>> I have created KIP-150 to facilitate discussion about adding
> > cogroup
> > > to
> > > >>>>> the streams DSL.
> > > >>>>>
> > > >>>>> Please find the KIP here:
> > > >>>>> https://cwiki.apache.org/confluence/display/KAFKA/KIP-
> > > >>>>> 150+-+Kafka-Streams+Cogroup
> > > >>>>>
> > > >>>>> Please find my initial implementation here:
> > > >>>>> https://github.com/KyleWinkelman/kafka
> > > >>>>>
> > > >>>>> Thanks,
> > > >>>>> Kyle Winkelman
> > > >>>>>
> > > >>>>
> > > >>
> > > >>
> > > >
> > >
> > >
> >
>
look
> > at
> > >>> the KIP yet, but will schedule some time to look into it tomorrow.
> For
> > >> the
> > >>> implementation, can you raise a PR against kafka trunk and mark it as
> > >> WIP?
> > >>> It will be
gt;> wrote:
> >>>
> >>>> I am replying to this in hopes it will draw some attention to my KIP
> as
> >> I
> >>>> haven't heard from anyone in a couple days. This is my first KIP and
> my
> >>>> first large contribution to the project so I'm sure I did something
> >> wrong.
> >>>> ;)
> >>>>
> >>>> On May 1, 2017 4:18 PM, "Kyle Winkelman"
> >> wrote:
> >>>>
> >>>>> Hello all,
> >>>>>
> >>>>> I have created KIP-150 to facilitate discussion about adding cogroup
> to
> >>>>> the streams DSL.
> >>>>>
> >>>>> Please find the KIP here:
> >>>>> https://cwiki.apache.org/confluence/display/KAFKA/KIP-
> >>>>> 150+-+Kafka-Streams+Cogroup
> >>>>>
> >>>>> Please find my initial implementation here:
> >>>>> https://github.com/KyleWinkelman/kafka
> >>>>>
> >>>>> Thanks,
> >>>>> Kyle Winkelman
> >>>>>
> >>>>
> >>
> >>
> >
>
>
> haven't heard from anyone in a couple days. This is my first KIP and my
>>>> first large contribution to the project so I'm sure I did something
>> wrong.
>>>> ;)
>>>>
>>>> On May 1, 2017 4:18 PM, "Kyle Winkelman"
>>
ng
> wrong.
> >> ;)
> >>
> >> On May 1, 2017 4:18 PM, "Kyle Winkelman"
> wrote:
> >>
> >>> Hello all,
> >>>
> >>> I have created KIP-150 to facilitate discussion about adding cogroup to
> >>> the str
GitHub user KyleWinkelman opened a pull request:
https://github.com/apache/kafka/pull/2975
KIP-150 [WIP]: Kafka Streams Cogroup
Work in progress PR for KIP-150.
You can merge this pull request into a Git repository by running:
$ git pull https://github.com/KyleWinkelman/kafka
t;> Hello all,
>>>
>>> I have created KIP-150 to facilitate discussion about adding cogroup to
>>> the streams DSL.
>>>
>>> Please find the KIP here:
>>> https://cwiki.apache.org/confluence/display/KAFKA/KIP-
>>> 150+-+Kafka-Streams+Cogroup
>>>
>>> Please find my initial implementation here:
>>> https://github.com/KyleWinkelman/kafka
>>>
>>> Thanks,
>>> Kyle Winkelman
>>>
>>
wrong.
> ;)
>
> On May 1, 2017 4:18 PM, "Kyle Winkelman" wrote:
>
> > Hello all,
> >
> > I have created KIP-150 to facilitate discussion about adding cogroup to
> > the streams DSL.
> >
> > Please find the KIP here:
> > https://cwiki.apac
; Hello all,
>
> I have created KIP-150 to facilitate discussion about adding cogroup to
> the streams DSL.
>
> Please find the KIP here:
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-
> 150+-+Kafka-Streams+Cogroup
>
> Please find my initial implementat
Hello all,
I have created KIP-150 to facilitate discussion about adding cogroup to the
streams DSL.
Please find the KIP here:
https://cwiki.apache.org/confluence/display/KAFKA/KIP-150+-+Kafka-Streams+Cogroup
Please find my initial implementation here:
https://github.com/KyleWinkelman/kafka
Kyle,
What's your apache id? I can grant you the permission.
Guozhang
On Sat, Apr 29, 2017 at 7:33 AM, Kyle Winkelman
wrote:
> I don't seem to have permission. When logged in I can neither edit the
> main page nor create an additional KIP.
>
> Thanks,
> Kyle
>
> On Thu, Apr 27, 2017 at 12:35
I don't seem to have permission. When logged in I can neither edit the main
page nor create an additional KIP.
Thanks,
Kyle
On Thu, Apr 27, 2017 at 12:35 PM, Eno Thereska
wrote:
> Hi Kyle,
>
> I believe Guozhang has now given you permission to edit the KIP wiki at
> https://cwiki.apache.org/con
Hi Kyle,
I believe Guozhang has now given you permission to edit the KIP wiki at
https://cwiki.apache.org/confluence/display/KAFKA/Kafka+Improvement+Proposals.
Could you see if you can add this there?
Many thanks
Eno
On Wed, Apr 26, 2017 at 6:00 PM, Kyle Winkelman
wrote:
> Thank you for your r
1 - 100 of 105 matches
Mail list logo