Re: [VOTE] SEP-26: Add SystemProducer for Azure Blob Storage

2020-01-08 Thread Prateek Maheshwari
+1 (binding). Thanks for the contribution.

- Prateek

On Tue, Jan 7, 2020 at 7:59 PM Jagadish Venkatraman 
wrote:

> +1 (binding), looking forward to Samza's integration with Azure blobs
>
> On Wednesday, January 8, 2020, Lakshmi Manasa 
> wrote:
>
> > Hi,
> >
> > This is a call for a vote on SEP-26: Add SystemProducer for Azure Blob
> > Storage.
> > Thanks for taking a look and giving feedback.
> >
> > I have addressed the comments on the SEP and since there were no major
> > questions/objections, starting this vote.
> >
> > Discussion thread:
> > http://mail-archives.apache.org/mod_mbox/samza-dev/202001.
> > mbox/%3CCAEwD47cW2T24C9A_tzj7Qxuv3P%2B2an47GkmaA4-
> > 41WZfvY_vgw%40mail.gmail.com%3E
> >
> > SEP:
> > https://cwiki.apache.org/confluence/display/SAMZA/SEP-
> > 26%3A+Azure+Blob+Storage+Producer
> >
> > Please vote:
> >
> > [ ] +1 approve
> >
> > [ ] +0 no opinion
> >
> > [ ] -1 disapprove (and reason why)
> >
> > Thanks,
> > Manasa
> >
>
>
> --
> Jagadish
>


Re: Reduce kafka partition for a topic samza is using

2020-01-08 Thread Bharath Kumara Subramanian
There are few ways to achieve data copy.
 1. Use the vanilla Kafka consumer that consumes data from the old topic
and produce to the new topic with fewer partitions.
 2. Write a Samza job that reads from your old topic and funnels the data
to the new topic.

I'd recommend you to follow up with Kafka community too if you are looking
for more options.

You typically don't have to delete the checkpoint and coordinator topic.
The checkpoints of the new tasks should overwrite the old tasks. However,
you might be left with some stale data since the new topic has fewer
partitions and hence fewer tasks. Coordinator topic stores config for the
most part and it is possible the topics have some stale topic
configurations (if any).

Hope that helps.

Thanks,
Bharath


On Tue, Jan 7, 2020 at 6:19 AM Debraj Manna 
wrote:

> Thanks Bharath for replying.
>
> Samza job is stateless and running in YARN cluster.
>
> If I follow the below approach.
>
>1. Create a temp kafka topic
>2. Copy the messages from old topic to the new topic
>3. Delete old topic
>4. Create new topic with required partitions
>5. Delete old topic
>6. Copy messages from temp topic to new topic
>
> What I have do with the __samza_checkpoint and __samza_coordinator_ topics?
> Should I also delete them?
>
> Can you explain what do you mean by reroute?
>
> On Mon, Jan 6, 2020 at 7:40 PM Bharath Kumara Subramanian <
> codin.mart...@gmail.com> wrote:
>
> > Hi Debraj,
> >
> > Kafka doesn't support reducing the partition size and only supports
> > increasing the partition size of a topic.
> > One way to accomplish it would be to create a new topic with the desired
> > partition count and reroute data from the old topic. Although, it will be
> > good to first understand the use case behind your request. Can you
> > shed some light on this?
> >
> > In the event of change to input topic partition count, the implications
> to
> > a Samza job are as follows
> >
> >1. For stateless jobs, the job is shutdown and if you are running in a
> >cluster mode (YARN), typically containers get restarted and pick up
> the
> >change. In case of Standalone, a new rebalance is triggered.
> >2. For stateful jobs, the shutdown behavior is the same. However,
> based
> >on the choice of the grouper, it might result in additional tasks or
> >reduced number of tasks which would invalidate some of the state
> > associated
> >with the tasks. If you have changelog enabled, you might need to
> > recreate
> >the changelog topic otherwise, you might run into validation issues or
> >correctness issues with your application.
> >
> > As of how Samza detects partition count changes[1] and the actions it
> takes
> > can be found here[2]
> >
> > Thanks,
> > Bharath
> >
> >
> > [1] -
> >
> >
> https://github.com/apache/samza/blob/master/samza-core/src/main/java/org/apache/samza/coordinator/StreamPartitionCountMonitor.java
> > [2] -
> >
> >
> https://github.com/apache/samza/blob/beb5e1b40c07c092bc6e14aafc131d96eda5fcd4/samza-core/src/main/java/org/apache/samza/clustermanager/ClusterBasedJobCoordinator.java#L370
> >
> >
> >
> > On Mon, Jan 6, 2020 at 4:31 AM Debraj Manna 
> > wrote:
> >
> > > Anyone any thoughts on this?
> > >
> > > On Sat, Jan 4, 2020 at 5:16 PM Debraj Manna 
> > > wrote:
> > >
> > > > I am using samza on yarn with Kafka. I need to reduce the number of
> > > > partitions in kafka. I am ok with some data loss. Can someone suggest
> > > what
> > > > should be the recommended way of doing this?
> > > >
> > > > Samza Job Config looks like this -
> > > >
> > > > job.factory.class = org.apache.samza.job.yarn.YarnJobFactory
> > > > task.class = com.vnera.grid.task.GenericStreamTask
> > > > task.window.ms = 100
> > > > systems.kafka.samza.factory =
> > > > org.apache.samza.system.kafka.KafkaSystemFactory
> > > > systems.kafka.consumer.zookeeper.connect = localhost:2181
> > > > systems.kafka.consumer.auto.offset.reset = largest
> > > > systems.kafka.producer.metadata.broker.list = localhost:9092
> > > > systems.kafka.producer.producer.type = sync
> > > > systems.kafka.producer.batch.num.messages = 1
> > > > systems.kafka.samza.key.serde = string
> > > > serializers.registry.string.class =
> > > > org.apache.samza.serializers.StringSerdeFactory
> > > > yarn.package.path =
> > > >
> > file://${basedir}/target/${project.artifactId}-${pom.version}-dist.tar.gz
> > > > yarn.container.count = ${container.count}
> > > > yarn.am.container.memory.mb =  ${samzajobs.memory.mb}
> > > > job.name = job4
> > > > task.inputs = kafka.Topic3
> > > >
> > >
> >
>


Re: Problem: upgrade 1.2 to 1.3 caused loss of clean shutdown on SIGTERM

2020-01-08 Thread Thunder Stumpges
Thanks Abhishek,

I believe you are correct; removing the shutdown hook from inside the
container was the problem. I took the shutdown hook code removed from
SamzaContainer in your commit #83e152904ef5 and pulled it out to our App
Runner, calling LocalApplicationRunner.kill() and then
LocalApplicationRunner.waitForFinish(timeout) and I think that has restored
all of the shutdown sequencing.

Where do you think this belongs (in the scope of SAMZA-2426 that you
created)? Maybe in LocalApplicationRunner itself? You said you already took
care of shutdown sequence in cluster mode, yes? I'm open to helping on this
one, just let me know.

thanks,
Thunder



On Tue, Jan 7, 2020 at 7:12 PM Abhishek S  wrote:

> The rational behind moving the shutdown handler out of SamzaContainer was
> to let standalone jobs (which are usually part of other online
> applications) maintain their own shutdown hooks.
> This prevents Samza hooks from running in parallel or causing a deadlock
> with the shutdown hooks of parent application that uses Samza in standalone
> mode (as a library).
>
> That being said, I agree that Containers should attempt graceful shutdown
> and wait at-most "task.shutdown.ms" on SIGTERM.
> I created SAMZA-2426 to investigate the issue further and track the work
> required.
>
> Abhishek
>
>
>
>
>
>
> On Tue, Jan 7, 2020 at 1:55 PM Brett Konold 
> wrote:
>
> > Thunder,
> >
> > How were you able to determine that the shutdown hooks are not being
> > called?
> >
> > If you're able to share any of your shutdown logs from before and after
> > your 1.3 upgrade, that would be helpful while I try to reproduce this
> issue
> > myself.
> >
> > Brett
> > 
> > From: Brett Konold 
> > Sent: Tuesday, January 7, 2020 1:25 PM
> > To: dev@samza.apache.org 
> > Subject: Re: Problem: upgrade 1.2 to 1.3 caused loss of clean shutdown on
> > SIGTERM
> >
> > Hey Thunder,
> >
> > Thanks for reporting. Taking a look into this and will get back to you
> > when I have something.
> >
> > Brett
> > 
> > From: Thunder Stumpges 
> > Sent: Monday, January 6, 2020 6:54 PM
> > To: dev@samza.apache.org 
> > Subject: Problem: upgrade 1.2 to 1.3 caused loss of clean shutdown on
> > SIGTERM
> >
> > We are attempting to upgrade from samza 1.2 to 1.3 in hopes of fixing
> >
> >
> https://nam06.safelinks.protection.outlook.com/?url=https%3A%2F%2Fissues.apache.org%2Fjira%2Fbrowse%2FSAMZA-2198data=02%7C01%7Cbkonold%40linkedin.com%7C3d5e0f0bf97e4cc633de08d7931d06ad%7C72f988bf86f141af91ab2d7cd011db47%7C1%7C0%7C637139625187723288sdata=uwKMfKxQmS9uWS62Hew814P%2Fja7FpaViNTVzYrp03Ok%3Dreserved=0
> > where there was a deadlock
> > in the shutdown code which prevented completing a clean shutdown.
> >
> > After the upgrade, it appears like NONE of the shutdown hooks / code are
> > being called and it just immediately shuts down.
> >
> > We are running stand alone Low Level Tasks with LocalApplicationRunner /
> > ZkJobCoordinator in Docker / K8s.
> >
> > When killing from docker for testing, we use docker kill -s SIGTERM
> >  to send SIGTERM instead of SIGKILL. This was working in samza
> > 1.2 (other than the deadlock from the above issue).
> >
> > Any ideas what changed?
> >
> > Thanks,
> > Thunder
> >
>


Re: Problem: upgrade 1.2 to 1.3 caused loss of clean shutdown on SIGTERM

2020-01-08 Thread Thunder Stumpges
Two things we have done here.

When troubleshooting the shutdown deadlock issue, we ran the app in docker
container with remote debugging and I could step through all of the Samza
(and our) related shutdown code. After the upgrade, none of the breakpoints
are hit. Not even our own shutdownHook nor spring context shutdownHook.

As for logs, before upgrade we saw all shutdown related logging, after
upgrade there is absolutely 0 logs. The JVM just immediately stops.

Thanks in advance. I'll have a look at the ticket also.

Thunder


On Tue, Jan 7, 2020 at 1:55 PM Brett Konold 
wrote:

> Thunder,
>
> How were you able to determine that the shutdown hooks are not being
> called?
>
> If you're able to share any of your shutdown logs from before and after
> your 1.3 upgrade, that would be helpful while I try to reproduce this issue
> myself.
>
> Brett
> 
> From: Brett Konold 
> Sent: Tuesday, January 7, 2020 1:25 PM
> To: dev@samza.apache.org 
> Subject: Re: Problem: upgrade 1.2 to 1.3 caused loss of clean shutdown on
> SIGTERM
>
> Hey Thunder,
>
> Thanks for reporting. Taking a look into this and will get back to you
> when I have something.
>
> Brett
> 
> From: Thunder Stumpges 
> Sent: Monday, January 6, 2020 6:54 PM
> To: dev@samza.apache.org 
> Subject: Problem: upgrade 1.2 to 1.3 caused loss of clean shutdown on
> SIGTERM
>
> We are attempting to upgrade from samza 1.2 to 1.3 in hopes of fixing
>
> https://nam06.safelinks.protection.outlook.com/?url=https%3A%2F%2Fissues.apache.org%2Fjira%2Fbrowse%2FSAMZA-2198data=02%7C01%7Cbkonold%40linkedin.com%7C3d5e0f0bf97e4cc633de08d7931d06ad%7C72f988bf86f141af91ab2d7cd011db47%7C1%7C0%7C637139625187723288sdata=uwKMfKxQmS9uWS62Hew814P%2Fja7FpaViNTVzYrp03Ok%3Dreserved=0
> where there was a deadlock
> in the shutdown code which prevented completing a clean shutdown.
>
> After the upgrade, it appears like NONE of the shutdown hooks / code are
> being called and it just immediately shuts down.
>
> We are running stand alone Low Level Tasks with LocalApplicationRunner /
> ZkJobCoordinator in Docker / K8s.
>
> When killing from docker for testing, we use docker kill -s SIGTERM
>  to send SIGTERM instead of SIGKILL. This was working in samza
> 1.2 (other than the deadlock from the above issue).
>
> Any ideas what changed?
>
> Thanks,
> Thunder
>