Hi Navinder,

Thanks for the response. I’m sorry if I’m being dense... You said we are not 
currently using the config, but I thought we would pass the config through to 
the client.  Can you confirm whether or not the existing config works for your 
use case?

Thanks,
John

On Sun, Jun 28, 2020, at 14:09, Navinder Brar wrote:
> Sorry my bad. Found it.
> 
> 
> 
> Prefix used to override {@link KafkaConsumer consumer} configs for the 
> global consumer client from
> 
> * the general consumer client configs. The override precedence is the 
> following (from highest to lowest precedence):
> * 1. global.consumer.[config-name]..
> public static final String GLOBAL_CONSUMER_PREFIX = "global.consumer.";
> 
> 
> 
> So, that's great. We already have a config exposed to reset offsets for 
> global topics via global.consumer.auto.offset.reset just that we are 
> not actually using it inside GlobalStreamThread to reset.
> 
> -Navinder
>     On Monday, 29 June, 2020, 12:24:21 am IST, Navinder Brar 
> <navinder_b...@yahoo.com.invalid> wrote:  
>  
>  Hi John,
> 
> Thanks for your feedback. 
> 1. I think there is some confusion on my first point, the enum I am 
> sure we can use the same one but the external config which controls the 
> resetting in global stream thread either we can the same one which 
> users use for source topics(StreamThread) or we can provide a new one 
> which specifically controls global topics. For e.g. currently if I get 
> an InvalidOffsetException in any of my source topics, I can choose 
> whether to reset from Earliest or Latest(with auto.offset.reset). Now 
> either we can use the same option and say if I get the same exception 
> for global topics I will follow same resetting. Or some users might 
> want to have totally different setting for both source and global 
> topics, like for source topic I want resetting from Latest but for 
> global topics I want resetting from Earliest so in that case adding a 
> new config might be better.
> 
> 2. I couldn't find this config currently 
> "global.consumer.auto.offset.reset". Infact in GlobalStreamThread.java 
> we are throwing a StreamsException for InvalidOffsetException and there 
> is a test as 
> well GlobalStreamThreadTest#shouldDieOnInvalidOffsetException(), so I 
> think this is the config we are trying to introduce with this KIP.
> 
> -Navinder  On Saturday, 27 June, 2020, 07:03:04 pm IST, John Roesler 
> <j...@vvcephei.org> wrote:  
>  
>  Hi Navinder,
> 
> Thanks for this proposal!
> 
> Regarding your question about whether to use the same policy
> enum or not, the underlying mechanism is the same, so I think
> we can just use the same AutoOffsetReset enum.
> 
> Can you confirm whether setting the reset policy config on the
> global consumer currently works or not? Based on my reading
> of StreamsConfig, it looks like it would be:
> "global.consumer.auto.offset.reset".
> 
> If that does work, would you still propose to augment the
> Java API?
> 
> Thanks,
> -John
> 
> On Fri, Jun 26, 2020, at 23:52, Navinder Brar wrote:
> > Hi,
> > 
> > KIP: 
> > https://cwiki.apache.org/confluence/display/KAFKA/KIP-406%3A+GlobalStreamThread+should+honor+custom+reset+policy
> > 
> > I have taken over this KIP since it has been dormant for a long time 
> > and this looks important for use-cases that have large global data, so 
> > rebuilding global stores from scratch might seem overkill in case of 
> > InvalidOffsetExecption.
> > 
> > We want to give users the control to use reset policy(as we do in 
> > StreamThread) in case they hit invalid offsets. I have still not 
> > decided whether to restrict this option to the same reset policy being 
> > used by StreamThread(using auto.offset.reset config) or add another 
> > reset config specifically for global stores 
> > "global.auto.offset.reset" which gives users more control to choose 
> > separate policies for global and stream threads.
> > 
> > I would like to hear your opinions on the KIP.
> > 
> > 
> > -Navinder

Reply via email to