Re: [DISCUSS] Correct time-related function behavior in Flink SQL

2021-01-20 Thread Jark Wu
Great examples to understand the problem and the proposed changes, @Kurt! Thanks Leonard for investigating this problem. The time-zone problems around time functions and windows have bothered a lot of users. It's time to fix them! The return value changes sound reasonable to me, and keeping the r

Re: Handling validations/errors in the flink job

2021-01-20 Thread sagar
Hi Team, any answer for my below question? On Wed, Jan 20, 2021 at 9:20 PM sagar wrote: > Hi Team, > > > I am creating a flink job with DataStream API and batch mode. > > It is having 5 different bounded sources and I need to perform some > business operations on it like joining , aggregating et

A few questions about minibatch

2021-01-20 Thread Rex Fenley
Hi, Our job was experiencing high write amplification on aggregates so we decided to give mini-batch a go. There's a few things I've noticed that are different from our previous job and I would like some clarification. 1) Our operators now say they have Watermarks. We never explicitly added water

Re: [DISCUSS] Correct time-related function behavior in Flink SQL

2021-01-20 Thread Kurt Young
cc this to user & user-zh mailing list because this will affect lots of users, and also quite a lot of users were asking questions around this topic. Let me try to understand this from user's perspective. Your proposal will affect five functions, which are: - PROCTIME() - NOW() - CURREN

Re: Flink Jobmanager HA deployment on k8s

2021-01-20 Thread Yang Wang
Hi Amit Bhatia > What is the correct way to start three jobmanager replicas with zk? Is there any link which explains this deployment scenario and configuration? Please find more information in the last mail. Unfortunately, we do not have some documentation to guide the users how to achieve that.

Re: Flink Jobmanager HA deployment on k8s

2021-01-20 Thread Yang Wang
Hi Chirag Dewan, Yes, we could have multiple replicas with ZK HA in K8 as well. Multiple JobManagers will contend for a leader and then write its rpc address to the ZooKeeper nodes. You could find more information how the HA service works here[1]. It is about the KubernetesHAService, but the ZooKe

Re: Kubernetes HA Services - artifact for KubernetesHaServicesFactory

2021-01-20 Thread Yang Wang
You do not need to put flink-kubernetes_2.12-1.12.1.jar under the plugin directory. Only the S3 fs jar needs to be put there. I think I found the root cause. It seems your flink-dist is still 1.11. Do you want to use the image 1.11 and enable the K8s HA at the same time? I think it could not work.

Re: Converting non-mini-batch to mini-batch from checkpoint or savepoint

2021-01-20 Thread Rex Fenley
Just tested this and I couldn't restore from a savepoint. If I do a new job from scratch, can I tune the minibatch parameters and restore from a savepoint without having to make yet another brand new job? Thanks On Wed, Jan 20, 2021 at 12:43 PM Rex Fenley wrote: > Hello, > > Is it safe to conv

Re: Kubernetes HA Services - artifact for KubernetesHaServicesFactory

2021-01-20 Thread Ashish Nigam
I did move to 1.12.1 version and also ensured that docker has kubernetes jar file in the right location, i.e. /opt/flink/plugins/s3-fs-presto/flink-kubernetes_2.12-1.12.1.jar But job manager is still not able to find the class 2021-01-21 00:00:49,376 ERROR org.apache.flink.runtime.entrypoint.Clus

org.apache.flink.runtime.client.JobSubmissionException: Job has already been submitted

2021-01-20 Thread Hailu, Andreas [Engineering]
Hello, We're running 1.9.2 on YARN, and are seeing some interesting behavior when submitting jobs in a multi-threaded fashion to an application's Flink cluster. The error we see reported in the client application logs is the following: org.apache.flink.client.program.ProgramInvocationException:

Converting non-mini-batch to mini-batch from checkpoint or savepoint

2021-01-20 Thread Rex Fenley
Hello, Is it safe to convert a non-mini-batch job to a mini-batch job when restoring from a checkpoint or a savepoint? Thanks -- Rex Fenley | Software Engineer - Mobile and Backend Remind.com | BLOG | FOLLOW US

Re: Problem with overridden hashCode/equals in keys in Flink 1.11.3 when checkpointing with RocksDB

2021-01-20 Thread David Haglund
I have an update. I have created a small project on github, https://github.com/daha/flink-key-by-problem-with-rocksdb-state, which reproduces the issue. There seems to be problem with RocksDB in all versions I have tested (from 1.7.1 and later). In Flink 1.9.x only one of the events is counted

Re: Restoring from a checkpoint or savepoint on a different Kafka consumer group

2021-01-20 Thread Rex Fenley
Thanks! On Tue, Jan 19, 2021 at 9:47 PM Piotr Nowojski wrote: > Hi Rex, > > Sorry, I might have misled you. I think you were right in your previous > email > > > So from the sounds of things, regardless of the consumer group's > offsets, it will always start from a checkpoint or savepoints offse

Re: question about timers

2021-01-20 Thread Seth Wiesman
Yes, Processing time timers that should have fired will fire immediately in order. Event time timers are never *late*, they will just fire when the watermark advances. Seth On Tue, Jan 19, 2021 at 3:58 PM Marco Villalobos wrote: > If there are timers that have been checkpointed (we use rocksd

Re: Trying to create a generic aggregate UDF

2021-01-20 Thread Dylan Forciea
Timo, I converted what I had to Java, and ended up with the exact same issue as before where it will work if I only ever use it on 1 type, but not if I use it on multiple. Maybe this is a bug? Dylan On 1/20/21, 10:06 AM, "Dylan Forciea" wrote: Oh, I think I might have a clue as to what

Re: Trying to create a generic aggregate UDF

2021-01-20 Thread Dylan Forciea
Oh, I think I might have a clue as to what is going on. I notice that it will work properly when I only call it on Long. I think that it is using the same generated code for the Converter for whatever was called first. Since in Scala I can't declare an object as static within the class itself, I

Handling validations/errors in the flink job

2021-01-20 Thread sagar
Hi Team, I am creating a flink job with DataStream API and batch mode. It is having 5 different bounded sources and I need to perform some business operations on it like joining , aggregating etc. I am using a CoGroup operator to join two streams as it serves as a left join. So when keys are

Re: Counter metrics for prometheus having unexepcted gaps in grafana

2021-01-20 Thread Chesnay Schepler
Not sure whether it would solve your issue, but you could maybe exclude the pod id (I assume you mean the host?) from being reported by setting this: metrics.reporter..scope.variables.excludes: On 1/20/2021 7:16 AM, Manish G wrote: Hi All, I am facing an issue with counter metrics I have a

Re: Trying to create a generic aggregate UDF

2021-01-20 Thread Dylan Forciea
As a side note, I also just tried to unify into a single function registration and used _ as the type parameter in the classOf calls there and within the TypeInference definition for the accumulator and still ended up with the exact same stack trace. Dylan On 1/20/21, 9:22 AM, "Dylan Forciea"

Re: Trying to create a generic aggregate UDF

2021-01-20 Thread Dylan Forciea
Timo, I appreciate it! I am using Flink 1.12.0 right now with the Blink planner. What you proposed is roughly what I had come up with the first time around that resulted in the stack trace with the ClassCastException I had originally included. I saw that you had used a Row instead of just the v

Re: Publishing a table to Kafka

2021-01-20 Thread Abhishek Rai
Thanks Leonard, we are working towards 1.12 upgrade and should be able to try upsert-kafka after that. > Your first workaround should have been worked, but looks like an exception > was thrown in Type conversion phase, could you share you table schema and > query that can reproduce the issue. I

Re: Error querying flink state

2021-01-20 Thread Till Rohrmann
Hi Falak, it is hard to tell what is going wrong w/o the debug logs. Could you check whether they contain anything specific? You can also share them with us. Cheers, Till On Wed, Jan 20, 2021 at 1:04 PM Falak Kansal wrote: > Hi, > > Thank you so much for the response. I am using the 1.12 versi

Problem with overridden hashCode/equals in keys in Flink 1.11.3 when checkpointing with RocksDB

2021-01-20 Thread David Haglund
I have encountered a problem in Flink when trying to upgrade from Flink 1.9.1 to Flink 1.11.3. The problem in a combination of 2 components: * Keys implemented as case classes in Scala where we override the equals and hashCode methods. The case class has additional fields which we are not used

Re: Trying to create a generic aggregate UDF

2021-01-20 Thread Timo Walther
Hi Dylan, I'm assuming your are using Flink 1.12 and the Blink planner? Beginning from 1.12 you can use the "new" aggregate functions with a better type inference. So TypeInformation will not be used in this stack. I tried to come up with an example that should explain the rough design. I wi