Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-07-09 Thread Matthias J. Sax
>>>>>>>>> >>>>>>>>> >>>>>>>> Sure, it is currently called withLogConfig() as that is the only >>>>>>>> thing >>>>>>> that >>>>>>>> is really config. &

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-07-07 Thread Matthias J. Sax
>>>>>>> it comes after the count() >>>>>>> >>>>>>> This would be more consistent than your proposal (that has grouping >>>>>>> before but windowing after count()). It might even allow us to >>>>

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-07-07 Thread Guozhang Wang
> > Sure, a generic name might be ok. >> > > >> > > >> > > >> > > >> > >> -Matthias >> > >> >> > >> >> > >> >> > >> On 6/29/17 7:37 AM, Damian Guy wrote: >&

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-07-07 Thread Guozhang Wang
> > legal > > >> to > > >>>>>>>> reuse it or does this approach expect you to call grouped each > > time? > > >>>>>> I'd anticipate that once you have a KGroupedStream you can re-use > it > > >> as

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-07-04 Thread Damian Guy
ontinue returning this (this being the > >>>> builder object currently being manipulated). > >>> So, if you ever store a reference to anything but KStreams and KTables > >> and > >>>> you use it in two different ways then its possible you m

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-07-04 Thread Jan Filipiak
re” already exists. Yes i agree and i think using builders is my preferred pattern. Cheers, Damian Thanks, Kyle From: Damian Guy Sent: Thursday, June 29, 2017 3:59 AM To: dev@kafka.apache.org Subject: Re: [DISCUSS] Streams DSL/StateStore Refactoring Hi Kyle, Thanks for your input. Really

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-30 Thread Matthias J. Sax
gt; to >>>>>>>>>> reuse it or does this approach expect you to call grouped each >> time? >>>>>> >>>>>>>> I'd anticipate that once you have a KGroupedStream you can re-use it >>>> as >>>>>&g

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-30 Thread Damian Guy
it > >>>> you wouldn’t be able to reuse anything except KStreams and KTables, > >> because > >>>> most of this fluent api would continue returning this (this being the > >>>> builder object currently being manipulated). > >>> > >>> So, if y

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-30 Thread Matthias J. Sax
o the builder >> it >>> would always be the same as it was when it was created. >>> >>> >>>> GroupedStream groupedStreamWithDefaultSerdes = kStream.grouped(); >>>> GroupedStream groupedStreamWithDeclaredSerdes = >>>> grouped

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-30 Thread Damian Guy
peated > >> processors. > >> > >> The point of this long winded example is that we always have to be > >> thinking about all of the possible ways it could be misused by a user > >> (causing them to see hard to diagnose problems). > >> > > >

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-30 Thread Matthias J. Sax
I feel that I could >> confidently say the user couldn’t really mess it up. >>> // Count >>> KTable count = >>> kGroupedStream.count(Count.count().withQueryableStoreName("my-store")); >> The kGroupedStream is reusable and if they attempted to reuse t

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-29 Thread Damian Guy
would throw an error message saying that a store named > “my-store” already exists. > > Yes i agree and i think using builders is my preferred pattern. Cheers, Damian > Thanks, > Kyle > > From: Damian Guy > Sent: Thursday, June 29, 2017 3:59 AM > To: dev@kafka.apache

RE: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-29 Thread Kyle Winkelman
le and if they attempted to reuse the Count for some reason it would throw an error message saying that a store named “my-store” already exists. Thanks, Kyle From: Damian Guy Sent: Thursday, June 29, 2017 3:59 AM To: dev@kafka.apache.org Subject: Re: [DISCUSS] Streams DSL/StateStore Refactoring Hi

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-29 Thread Damian Guy
I've updated the experimental code with a couple of ways of doing joins. One following the fluent approach and one following the builder approach. The 2 examples can be found here: https://github.com/dguy/kafka/blob/dsl-experiment/streams/src/test/java/org/apache/kafka/streams/kstream/internals/KSt

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-29 Thread Damian Guy
> However, I don't understand your argument about putting aggregate() > after the withXX() -- all the calls to withXX() set optional parameters > for aggregate() and not for groupBy() -- but a groupBy().withXX() > indicates that the withXX() belongs to the groupBy(). IMHO, this might > be quite con

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-29 Thread Damian Guy
> public KTable count(Count count); > public KTable, Long> count(Count.Windowed count); > public KTable, Long> count(Count.SessionWindowed count); > … > } > > > Thanks, > Kyle > > From: Guozhang Wang > Sent: Wednesday, June 28, 2017 7:45 PM > To: de

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-29 Thread Damian Guy
Hi Guozhang, Thanks for the feedback On Thu, 29 Jun 2017 at 01:45 Guozhang Wang wrote: > I played the current proposal a bit with https://github.com/dguy/kafka/ > tree/dsl-experiment , > and here are my observations: > > 1. Personally I prefer

RE: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-28 Thread Kyle Winkelman
esday, June 28, 2017 7:45 PM To: dev@kafka.apache.org Subject: Re: [DISCUSS] Streams DSL/StateStore Refactoring I played the current proposal a bit with https://github.com/dguy/kafka/ tree/dsl-experiment, and here are my observations: 1. Personally I prefer "stream.group(mapper)

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-28 Thread Guozhang Wang
I played the current proposal a bit with https://github.com/dguy/kafka/ tree/dsl-experiment, and here are my observations: 1. Personally I prefer "stream.group(mapper) / stream.groupByKey()" than "stream.group().withKeyMapper(mapper) / stream.group()" Since 1) withKeyMapper is not enfo

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-28 Thread Matthias J. Sax
I see your point about "when to add the processor to the topology". That is indeed an issue. Not sure it we could allow "updates" to the topology... I don't see any problem with having all the withXX() in KTable interface -- but this might be subjective. However, I don't understand your argument

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-28 Thread Damian Guy
> I also think that mixing optional parameters with configs is a bad idea. > Have not proposal for this atm but just wanted to mention it. Hope to > find some time to come up with something. > > Yes, i don't like the mix of config either. But the only real config here is the logging config - which

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-24 Thread Matthias J. Sax
Damian, thanks for starting this discussion. I am not a fan of the builder pattern. It's too clumsy to use IMHO an raised the entry level bar. I also think that mixing optional parameters with configs is a bad idea. Have not proposal for this atm but just wanted to mention it. Hope to find some

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-22 Thread Damian Guy
Thanks everyone. My latest attempt is below. It builds on the fluent approach, but i think it is slightly nicer. I agree with some of what Eno said about mixing configy stuff in the DSL, but i think that enabling caching and enabling logging are things that aren't actually config. I'd probably not

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-22 Thread Ismael Juma
Thanks Damian. I think both options have pros and cons. And both are better than overload abuse. The fluent API approach reads better, no mention of builder or build anywhere. The main downside is that the method signatures are a little less clear. By reading the method signature, one doesn't nece

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-22 Thread Jan Filipiak
Hi Eno, I am less interested in the user facing interface but more in the actual implementation. Any hints where I can follow the discussion on this? As I still want to discuss upstreaming of KAFKA-3705 with someone Best Jan On 21.06.2017 17:24, Eno Thereska wrote: (cc’ing user-list too)

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-22 Thread Eno Thereska
Note that while I agree with the initial proposal (withKeySerdes, withJoinType, etc), I don't agree with things like .materialize(), .enableCaching(), .enableLogging(). The former maintain the declarative DSL, while the later break the declarative part by mixing system decisions in the DSL. I

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-21 Thread Eno Thereska
To make it clear, it’s outlined by Damian, I just copy pasted what he told me in person :) Eno > On Jun 21, 2017, at 4:40 PM, Bill Bejeck wrote: > > +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 cou

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: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-21 Thread Damian Guy
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 parameters and everything else is specified as you have done above. On Wed, 21 Jun 2017 at 16:24 Eno Ther

Re: [DISCUSS] Streams DSL/StateStore Refactoring

2017-06-21 Thread Eno Thereska
(cc’ing user-list too) Given that we already have StateStoreSuppliers that are configurable using the fluent-like API, probably it’s worth discussing the other examples with joins and serdes first since those have many overloads and are in need of some TLC. So following your example, I guess yo

[DISCUSS] Streams DSL/StateStore Refactoring

2017-06-21 Thread Damian Guy
Hi, I'd like to get a discussion going around some of the API choices we've made in the DLS. In particular those that relate to stateful operations (though this could expand). As it stands we lean heavily on overloaded methods in the API, i.e, there are 9 overloads for KGroupedStream.count(..)! It