Re: SystemProducer wants to call commit after flush

2015-01-29 Thread Chris Riccomini
Lol, we just came to the same conclusion. Can you open up a JIRA, so we can discuss there? I have some concerns about the new Kafka producer that we just upgraded to (see above). We can discuss more there. On Thu, Jan 29, 2015 at 10:13 AM, Bae, Jae Hyeon wrote: > What about this idea? It might s

Re: SystemProducer wants to call commit after flush

2015-01-29 Thread Chris Riccomini
Hey Jae, Yea, I've been thinking about this a bit. I think there's no good way to do this right now, but it seems like a reasonable request. I think the goal would be to trigger a commit every time a flush is called. This would MINIMIZE but not ELIMINATE the potential for duplicate messages. There

Re: SystemProducer wants to call commit after flush

2015-01-29 Thread Bae, Jae Hyeon
What about this idea? It might sound ugly and unnecessarily complicated, but message sending chain should have coordinator. MessageCollector.send(envelop, coordinator) SystemProducers.send(source, envelop, coordinator) SystemProducer.send(source, envelop, coordinator) If it sounds acceptable, I w

Re: SystemProducer wants to call commit after flush

2015-01-29 Thread Bae, Jae Hyeon
Hi Chris On Thu, Jan 29, 2015 at 9:10 AM, Chris Riccomini wrote: > Hey Jae, > > If I understand you correctly, your concern is that there could be flushes > in-between commits. For example: > > T=30s; flush > T=45s; flush > T=60s; flush && commit > T=65s; flush > > Your concern here is

Re: SystemProducer wants to call commit after flush

2015-01-29 Thread Chris Riccomini
Hey Jae, If I understand you correctly, your concern is that there could be flushes in-between commits. For example: T=30s; flush T=45s; flush T=60s; flush && commit T=65s; flush Your concern here is that if there's a failure before 60s, the messages that were flushed at 30s and 45s will

Re: SystemProducer wants to call commit after flush

2015-01-29 Thread Bae, Jae Hyeon
Never mind. I found a solution. Flush should be synced with commit. On Thu, Jan 29, 2015 at 12:15 AM, Bae, Jae Hyeon wrote: > Hi Samza Devs > > StreamTask can control SamzaContainer.commit() through task coordinator. > Can we make SystemProducer control commit after flush? With this feature, > w

SystemProducer wants to call commit after flush

2015-01-29 Thread Bae, Jae Hyeon
Hi Samza Devs StreamTask can control SamzaContainer.commit() through task coordinator. Can we make SystemProducer control commit after flush? With this feature, we can prevent any duplicate data on SamzaContainer failure. For example, if we set commit interval as 2 minutes, before commit time int