Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Mich Talebzadeh
Hi Jorn, Here you go the dependencies libraryDependencies += "org.apache.hadoop" % "hadoop-core" % "1.2.1" libraryDependencies += "org.apache.hbase" % "hbase" % "1.2.6" libraryDependencies += "org.apache.hbase" % "hbase-client" % "1.2.6" libraryDependencies += "org.apache.hbase" % "hbase-common"

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Jörn Franke
How does your build.sbt looks especially dependencies? > On 2. Aug 2018, at 00:44, Mich Talebzadeh wrote: > > Changed as suggested > >val streamExecEnv = StreamExecutionEnvironment.getExecutionEnvironment > val dataStream = streamExecEnv >.addSource(new

Re: Delay in REST/UI readiness during JM recovery

2018-08-01 Thread vino yang
Hi Joey, Currently rest endpoints are hosted in JM. Your scenario is at JM failover, and your cluster is running so many jobs. Here, it takes a certain amount of time for ZK to conduct the Leader election. Then JM needs to wait for the TM registration. So many jobs need to be restored and start

Re: Old job resurrected during HA failover

2018-08-01 Thread vino yang
Hi Elias, Your analysis is correct, yes, in theory the old jobgraph should be deleted, but Flink currently uses the method of locking and asynchronously deleting Path, so that it can not give you the acknowledgment of deleting, so this is a risk point. cc Till, there have been users who have

Re: Flink log and out files

2018-08-01 Thread vino yang
Hi Alexander, Yes, usually if you configure the log output as a .log file, the .out file will not have content, but sometimes some fatal exceptions, such as JVM exit exceptions, may be printed to the .out file. If Flink logs appear in your .out file, and it seems that the format of the log is

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread vino yang
Hi Mich, It seems that the type of your DataStream stream is always wrong. If you want to specify four fields, usually the DataStream type should be similar: DataStream[(Type1, Type2, Type3, Type4)], not DataStream[String], you can try it. Thanks, vino 2018-08-02 6:44 GMT+08:00 Mich Talebzadeh

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Mich Talebzadeh
Changed as suggested val streamExecEnv = StreamExecutionEnvironment.getExecutionEnvironment val dataStream = streamExecEnv .addSource(new FlinkKafkaConsumer011[String](topicsValue, new SimpleStringSchema(), properties)) val tableEnv =

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Fabian Hueske
Hi, You have to pass the StreamExecutionEnvironment to the getTableEnvironment() method, not the DataStream (or DataStreamSource). Change val tableEnv = TableEnvironment.getTableEnvironment(dataStream) to val tableEnv = TableEnvironment.getTableEnvironment(streamExecEnv) Best, Fabian

Re: Counting elements that appear "behind" the watermark

2018-08-01 Thread Julio Biason
Awesome, thanks Elias! On Tue, Jul 31, 2018 at 10:02 PM, Elias Levy wrote: > Correct. Context gives you access to the element timestamp > . > But it

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Mich Talebzadeh
Hi, FYI, these are my imports import java.util.Properties import java.util.Arrays import org.apache.flink.api.common.functions.MapFunction import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment import org.apache.flink.streaming.api.scala import

Re: Old job resurrected during HA failover

2018-08-01 Thread Elias Levy
I can see in the logs that the JM 1 (10.210.22.167), that one that became leader after failover, thinks it deleted the 2a4eff355aef849c5ca37dbac04f2ff1 job from ZK when it was canceled: July 30th 2018, 15:32:27.231 Trying to cancel job with ID 2a4eff355aef849c5ca37dbac04f2ff1. July 30th 2018,

Re: Delay in REST/UI readiness during JM recovery

2018-08-01 Thread Joey Echeverria
Sorry to ping my own thread, but has anyone else encountered this? -Joey > On Jul 30, 2018, at 11:10 AM, Joey Echeverria wrote: > > I’m running Flink 1.5.0 in Kubernetes with HA enabled, but only a single Job > Manager running. I’m using Zookeeper to store the fencing/leader information >

Re: Flink log and out files

2018-08-01 Thread Alexander Smirnov
thanks guys, So, is it a correct statement - if my job doesn't write anything to stdout, the "*.out" file should be empty? for some reason it contains the same info as "log" and much more. For the "log" files, I can control rotation via log4j configuration, but how do I setup rotation for "out"

Re: Old job resurrected during HA failover

2018-08-01 Thread Elias Levy
Vino, Thanks for the reply. Looking in ZK I see: [zk: localhost:2181(CONNECTED) 5] ls /flink/cluster_1/jobgraphs [d77948df92813a68ea6dfd6783f40e7e, 2a4eff355aef849c5ca37dbac04f2ff1] Again we see HA state for job 2a4eff355aef849c5ca37dbac04f2ff1, even though that job is no longer running (it

Re: Dynamical Windows

2018-08-01 Thread antonio saldivar
Awesome, thank you very much I will try to do it with key selector to send the key from the front end El mié., 1 ago. 2018 a las 11:57, vino yang () escribió: > Sorry, the KeySelector's Java doc is here : >

Re: Flink log and out files

2018-08-01 Thread Andrey Zagrebin
Hi Alexander, there is also a doc link where log configuration is described: https://ci.apache.org/projects/flink/flink-docs-release-1.5/monitoring/logging.html You can modify log configuration in conf

Re: Dynamical Windows

2018-08-01 Thread vino yang
Sorry, the KeySelector's Java doc is here : https://ci.apache.org/projects/flink/flink-docs-master/api/java/org/apache/flink/api/java/functions/KeySelector.html 2018-08-01 23:57 GMT+08:00 vino yang : > Hi antonio, > > The keyBy API can accept a KeySelector [1] which is a interface you can >

Re: Dynamical Windows

2018-08-01 Thread vino yang
Hi antonio, The keyBy API can accept a KeySelector [1] which is a interface you can implement to specify the key for your business. I think you can use it and implement its getKey method. In the method, you can access outer system (such as Zookeeper) to get a dynamic key. It's just an idea, you

Dynamical Windows

2018-08-01 Thread antonio saldivar
Hello I am developing a Flink 1.4.2 application currently with sliding windows (Example below) I want to ask if there is a way to create the window time dynamically also the key has to change in some Use Cases and we don't want to create an specific window for each UC I want to send those values

Re: Rest API calls

2018-08-01 Thread Rong Rong
Hi Yuvraj, Vino is right, having a customized function is probably the easiest at this moment. Alternatively, I think what you are looking for is very much similar to side-input feature of data stream[2]. Thanks, Rong [2] FLIP-17:

Re: Old job resurrected during HA failover

2018-08-01 Thread vino yang
Hi Elias, If a job is explicitly canceled, its jobgraph node on ZK will be deleted. However, it is worth noting here that Flink enables a background thread to asynchronously delete the jobGraph node, so there may be cases where it cannot be deleted. On the other hand, the jobgraph node on ZK is

Re: Flink log and out files

2018-08-01 Thread vino yang
Hi Alexander: .log and .out are different. Usually, the .log file stores the log information output by the log framework. Flink uses slf4j as the log interface and supports log4j and logback configurations. The .out file stores the STDOUT information. This information is usually output by you

Flink log and out files

2018-08-01 Thread Alexander Smirnov
Hi, could you please explain the difference between *.log and *.out files in Flink? What information is supposed to be in each of them? Is "log" a subset of "out"? How do I setup rotation with gzipping? Thank you, Alex

Old job resurrected during HA failover

2018-08-01 Thread Elias Levy
For the second time in as many months we've had an old job resurrected during HA failover in a 1.4.2 standalone cluster. Failover was initiated when the leading JM lost its connection to ZK. I opened FLINK-10011 with the details. We are using

Re: Could not retrieve the redirect address - No REST endpoint has been started

2018-08-01 Thread Andrey Zagrebin
Hi Pedro, when you restart the cluster, do you keep Web UI open? and does Web UI work eventually after restart and getting this error? Cheers, Andrey > On 1 Aug 2018, at 11:56, PedroMrChaves wrote: > > Hello, > > I have a running standalone Flink cluster with 2 task managers and 2 job >

Re: Access to Kafka Event Time

2018-08-01 Thread Hequn Cheng
Hi Vishal, > We have a use case where multiple topics are streamed to hdfs and we would want to created buckets based on ingestion time If I understand correctly, you want to create buckets based on event time. Maybe you can use window[1]. For example, a tumbling window of 5 minutes groups rows

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Fabian Hueske
Hi I think you are mixing Java and Scala dependencies. org.apache.flink.streaming.api.datastream.DataStream is the DataStream of the Java DataStream API. You should use the DataStream of the Scala DataStream API. Best, Fabian 2018-08-01 14:01 GMT+02:00 Mich Talebzadeh : > Hi, > > I believed I

Re: Access to Kafka Event Time

2018-08-01 Thread Vishal Santoshi
Any feedbaxk? On Tue, Jul 31, 2018, 10:20 AM Vishal Santoshi wrote: > In fact it may be available else where too ( for example ProcessFunction > etc ) but do we have no need to create one, it is just a data relay ( kafka > to hdfs ) and any intermediate processing should be avoided if possible

Re: Default Restart Strategy Not Work With Checkpointing

2018-08-01 Thread Chesnay Schepler
Please see FLINK-9143 . On 01.08.2018 14:07, Paul Lam wrote: Hi, I’m running a Flink 1.5.0 standalone cluster on which `restart-strategy` was set to `failure-rate`, and the web frontend shows that the JobManager and the TaskManagers are

Re: Rest API calls

2018-08-01 Thread vino yang
Hi yuvraj, Sorry, I didn't see it clearly. I think in many UDF contexts, such as MapFunction, ProcessFunction, etc., you can access the Rest API as a client. Also if you want to improve performance, maybe async I/O will help you[1]. [1]:

Default Restart Strategy Not Work With Checkpointing

2018-08-01 Thread Paul Lam
Hi, I’m running a Flink 1.5.0 standalone cluster on which `restart-strategy` was set to `failure-rate`, and the web frontend shows that the JobManager and the TaskManagers are following this configuration, but streaming jobs with checkpointing enabled are still using the fixed delay strategy

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Mich Talebzadeh
Hi, I believed I tried Hequn's suggestion and tried again import org.apache.flink.table.api.Table import org.apache.flink.table.api.TableEnvironment *import org.apache.flink.table.api.scala._* Unfortunately I am still getting the same error! [info] Compiling 1 Scala source to

Could not retrieve the redirect address - No REST endpoint has been started

2018-08-01 Thread PedroMrChaves
Hello, I have a running standalone Flink cluster with 2 task managers and 2 job manages (one task manager and job manager per machine). Sometimes, when I restart the cluster I get the following error message: / java.util.concurrent.CompletionException: org.apache.flink.util.FlinkException: No

Re: Service discovery for flink-metrics-prometheus

2018-08-01 Thread Dongwon Kim
Hi all, I also suffer from the lack of service discovery for flink-metrics-prometheus while using YARN for deployment, Prometheus for instrumentation, and Flink for stream processing. I just upload a Python script for the purpose here: https://github.com/eastcirclek/flink-service-discovery

Re: [ANNOUNCE] Apache Flink 1.5.2 released

2018-08-01 Thread Till Rohrmann
Great work! Thanks to everyone who helped with the release. Cheers, Till On Tue, Jul 31, 2018 at 7:43 PM Bowen Li wrote: > Congratulations, community! > > On Tue, Jul 31, 2018 at 1:44 AM Chesnay Schepler > wrote: > >> The Apache Flink community is very happy to announce the release of >>

Service discovery for Prometheus on YARN

2018-08-01 Thread Dongwon Kim
Hi community, Prior to my presentation [1], I'd like to share a Python script [2] to discover Flink clusters on YARN and let Prometheus know via its file-based service discovery mechanism [3]. Prometheus needs to pull metrics from Prometheus exporters running inside TaskManagers. The problem

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Timo Walther
If these two imports are the only imports that you added, then you did not follow Hequn's advice or the link that I sent you. You need to add the underscore imports to let Scala do its magic. Timo Am 01.08.18 um 10:28 schrieb Mich Talebzadeh: Hi Timo, These are my two flink table related

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Mich Talebzadeh
Hi Timo, These are my two flink table related imports import org.apache.flink.table.api.Table import org.apache.flink.table.api.TableEnvironment And these are my dependencies building with SBT libraryDependencies += "org.apache.hadoop" % "hadoop-core" % "1.2.1" libraryDependencies +=

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Timo Walther
Hi Mich, I would check you imports again [1]. This is a pure compiler issue that is unrelated to your actual data stream. Also check your project dependencies. Regards, Timo [1] https://ci.apache.org/projects/flink/flink-docs-master/dev/table/common.html#implicit-conversion-for-scala Am

Re: Converting a DataStream into a Table throws error

2018-08-01 Thread Mich Talebzadeh
Hi both, I added the import as Hequn suggested. My stream is very simple and consists of 4 values separated by "," as below 05521df6-4ccf-4b2f-b874-eb27d461b305,IBM,2018-07-30T19:51:50,190.48 So this is what I have been trying to do Code val dataStream = streamExecEnv

Re: AM Delegation Token Regeneration

2018-08-01 Thread Paul Lam
Hi Chen, Thanks for the quick reply! I’ve read the design document and it is very much what I’m looking for. And I think the design was absorbed in FLIP-26, right? I will keep watching this FLIP. Thanks again. Best regards, Paul Lam