Re: Flink 1.13 fails to load log4j2 yaml configuration file via jackson-dataformat-yaml

2021-07-29 Thread Yuval Itzchakov
It is finding the file though, the problem is that the lib/ might not be on the classpath when the file is being parsed, thus the YAML file is not recognized as being parsable. Is there a way to differ the inference from BashJavaUtils to the actual bootstrap of the app? or perhaps add the lib to t

Re: How can I declare BigDecimal precision and scale in user-defined aggregate function?

2021-07-29 Thread LIU Xiao
I actually have already specified the data type int, but that doesn't work: public void accumulate(LastDecimalAccumulator accumulator, > @DataTypeHint("DECIMAL(38, 18)") BigDecimal value) > { > if (value != null) { > accumulator.f0 = value; > } > } > I did

Re: How can I declare BigDecimal precision and scale in user-defined aggregate function?

2021-07-29 Thread Maciej Bryński
Hi, You can do sth like this: /** * UDF implementing Power function with Decimal */ public class PowerFunction extends ScalarFunction { public static MathContext mc = new MathContext(18); public @DataTypeHint("DECIMAL(38,18)") BigDecimal eval(@DataTypeHint("DECIMAL(38,18)") BigDecimal

Re: Obtain JobManager Web Interface URL

2021-07-29 Thread Yangze Guo
Hi, Hailu AFAIK, the ClusterClient#getWebInterfaceURL has been available since 1.10. Regarding the JobManager web interface, it will be print in the logs when staring a native Kubernetes or Yarn cluster. In standalone mode, it is configured by yourself[1]. [1] https://ci.apache.org/projects/fli

Re: How can I declare BigDecimal precision and scale in user-defined aggregate function?

2021-07-29 Thread LIU Xiao
sorry for a little error, the program code should be: package poc.flink.table; import org.apache.flink.api.common.eventtime.WatermarkStrategy; import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.typeinfo.TypeInformation; import org.apache.flink.api.common.

How can I declare BigDecimal precision and scale in user-defined aggregate function?

2021-07-29 Thread LIU Xiao
I'm currently converting our old code (based on Flink 1.6) to Flink 1.13 and encountered a strange problem about the user-defined aggregate function which takes BigDecimal as the parameter and output: Exception in thread "main" org.apache.flink.table.api.ValidationException: > SQL validation faile

Re: Migrating Kafka Sources (major version change)

2021-07-29 Thread Dan Hill
Are there any docs that talk about the new idleness support? I want to understand it better. https://ci.apache.org/projects/flink/flink-docs-release-1.13/docs/connectors/datastream/kafka/ On Thu, Jul 29, 2021 at 6:15 PM Dan Hill wrote: > Thanks, JING and Arvid! > > Interesting. That's good to

Re: Migrating Kafka Sources (major version change)

2021-07-29 Thread Dan Hill
Thanks, JING and Arvid! Interesting. That's good to know. I've been planning for incompatible schema changes. I'll look into new source too. On Thu, Jul 29, 2021 at 4:56 AM Arvid Heise wrote: > I'm assuming you have an incompatible schema change. If you don't, there > are several easy ways.

Obtain JobManager Web Interface URL

2021-07-29 Thread Hailu, Andreas [Engineering]
Hi team, Is there a method available to obtain the JobManager's REST url? We originally overloaded CliFrontend#executeProgram and nabbed it from the ClusterClient#getWebInterfaceUrl method, but it seems this method's signature has been changed and no longer available as of 1.10.0. Best, Andrea

Error using KafkaProducer EXACTLY_ONCE semantic + TaskManager Failure

2021-07-29 Thread Kevin Lam
Hi user@, We're developing a Flink application, and using the FlinkKafkaProducer. Semantic.EXACTLY_ONCE producer semantic to output records to a Kafka topic in an exactly-once way. We run our flink application on kubernetes. I've observed that if a task manager fails (I've simulated this by killi

Unable to use custom AWS credentials provider - 1.9.2

2021-07-29 Thread Hailu, Andreas [Engineering]
Hi team, I'm trying to read and write from and to S3 using a custom AWS Credential Provider using Flink v1.9.2 on YARN. I followed the instructions to create a plugins directory in our Flink distribution location and copy the FS implementation (I'm using s3-fs-hadoop) package into it. I have al

Re: Flink 1.13 fails to load log4j2 yaml configuration file via jackson-dataformat-yaml

2021-07-29 Thread Chesnay Schepler
That could be...could you try configuring "env.java.opts: -Dlog4j.configurationFile=..."? On 29/07/2021 15:18, Yuval Itzchakov wrote: Perhaps because I'm passing it as an argument to the job with -Dlog4j.configurationFile=... ? On Thu, Jul 29, 2021 at 3:27 PM Chesnay Schepler

Re: TaskManager crash after cancelling a job

2021-07-29 Thread Piotr Nowojski
Hi Ivan, It sounds to me like a bug in FlinkKinesisConsumer that it's not cancelling properly. The change in the behaviour could have been introduced as a bug fix [1], where we had to stop interrupting the source thread. This also might be related or at least relevant for fixing [2]. Ivan, the st

Re: Flink 1.13 fails to load log4j2 yaml configuration file via jackson-dataformat-yaml

2021-07-29 Thread Yuval Itzchakov
Perhaps because I'm passing it as an argument to the job with -Dlog4j.configurationFile=... ? On Thu, Jul 29, 2021 at 3:27 PM Chesnay Schepler wrote: > Looking once more at the stacktrace this happens in the BashJavaUtils, > which only puts flink-dist and not the entire lib/ directory on the > c

Re: Checkpoints question

2021-07-29 Thread Arvid Heise
Hi Kirill, Your understanding of checkpointing is very good. Just a couple of additions. Incremental checkpoints usually help when you have a low number of changed keys. Afaik in statefun, the keys correspond to a function address (name X id). So if you have few function addresses and all of them

when scalarfunction return treemap,the next operator is out of order

2021-07-29 Thread 张颖
I extends scalarfunction like this,and I invoke tableEnv.createTemporarySystemFunction("tree_map", new TreeMapFunction()); public class TreeMapFunction extends ScalarFunction { private static Logger profileLog = Logger.getLogger("zhangying480"); @Override public boolean isDeterministic() { return

Cleaning old incremental checkpoint files

2021-07-29 Thread Robin Cassan
Hi all! We've happily been running a Flink job in production for a year now, with the RocksDB state backend and incremental retained checkpointing on S3. We often release new versions of our jobs, which means we cancel the running one and submit another while restoring the previous jobId's last re

Re: Flink 1.13 fails to load log4j2 yaml configuration file via jackson-dataformat-yaml

2021-07-29 Thread Chesnay Schepler
Looking once more at the stacktrace this happens in the BashJavaUtils, which only puts flink-dist and not the entire lib/ directory on the classpath. What I don't understand is why it would pick up your log4j file. It should only use the file that is embedded within BashJavaUtils.jar. On 29/07

Re: Migrating Kafka Sources (major version change)

2021-07-29 Thread Arvid Heise
I'm assuming you have an incompatible schema change. If you don't, there are several easy ways. Your plan actually looks like the best option. Of course, this requires that you eventually union the inputs. If you can do that without a custom mapper and with one read schema only, you may even use 1

Re: Questions about keyed streams

2021-07-29 Thread Arvid Heise
Afaik you can express the partition key in Table API now which will be used for co-location and optimization. So I'd probably give that a try first and convert the Table to DataStream where needed. On Sat, Jul 24, 2021 at 9:22 PM Dan Hill wrote: > Thanks Fabian and Senhong! > > Here's an example

Re: YarnTaskExecutorRunner should contains MapReduce classes

2021-07-29 Thread Arvid Heise
Hi Chenkaibit, Let's further discuss it on the ticket as it indeed looks like a bug/missing feature. I think Cloudera is currently taking the lead of Flink-YARN development, so you will get a response soonish. On Fri, Jul 23, 2021 at 8:59 AM chenkaibit wrote: > > Hi: > I followed instructions d

Re: Need help of deploying Flink HA on kubernetes cluster

2021-07-29 Thread Fabian Paul
Hi Dhiru, Sorry for the late reply. Once the cluster is successfully started the web UI should be reachable if you somehow forward the port of the running pod. Although with the exception you have shared I suspect the cluster never fully runs (or not long enough). Can you share the full stacktra

Re: Flink 1.13 fails to load log4j2 yaml configuration file via jackson-dataformat-yaml

2021-07-29 Thread Yuval Itzchakov
Hi Chesnay, So I looked into it, and jackson-databind is definitely in my UBER jar as a transitive dependency. Additionally, this uber JAR is added to the `lib/` directly for Flink before I start the job, yet it still fails. Are there any other paths I should explore? One thing to add is that I'm