> On 13 Oct 2015, at 16:06, schul...@informatik.hu-berlin.de wrote:
>
> Hello,
>
> I am currently working on a compilation unit translating AsterixDB's AQL
> into runnable Scala code for Flink's Scala API. During code generation I
> discovered some things that are quite hard to work around. I am
> On 12 Oct 2015, at 22:47, Jerry Peng wrote:
>
> Hello,
>
> I am trying to do some benchmark testing with flink streaming. When flink
> reads a message in from Kafka, I want to write a timestamp to redis. How can
> I modify the existing kafka consumer code to do this? What would be easies
> On 10 Oct 2015, at 22:59, snntr wrote:
>
> Hey everyone,
>
> I was having the same problem with S3 and found this thread very useful.
> Everything works fine now, when I start Flink from my IDE, but when I run
> the jar in local mode I keep getting
>
> java.lang.IllegalArgumentException: A
Hey Michele! Sorry for the long delay on this. Can you share the code and
data? The easiest thing to reproduce is to actually run the code with the
data. It will also help a lot when trying to fix the problem. If you can't
share publicly feel free to mail me at uce at apache org.
– Ufuk
On Thu, O
Hey Jerry,
On Wed, Oct 21, 2015 at 11:11 PM, Jerry Peng
wrote:
>
> When I submit the job, the number of task slots that gets used
> (displayed on the UI) is only 20. Why is that? The total number of
> tasks listed on the ui is 55.
Do you mean the number of task slots is 55 (you just wrote task
Hey Nick,
you can do the following for per task stats (this is kind of an workaround):
Create an Accumulator with the subtask index in the name, e.g.
int subtaskIndex = getRuntimeContext().getIndexOfThisSubtask();
IntCounter counter = getRuntimeContext().getIntCounter("counter-" +
subtaskIndex)
> On 13 Nov 2015, at 15:49, LINZ, Arnaud wrote:
>
> Hi Robert,
>
> Thanks, it works with 50% -- at least way past the previous crash point.
>
> In my opinion (I lack real metrics), the part that uses the most memory is
> the M2 mapper, instantiated once per slot.
> The most complex part is
I’ve added a note about this to the docs and asked Max to trigger a new build
of them.
Regarding Aljoscha’s idea: I like it. It is essentially a shortcut for
configuring the root path.
In any case, it is orthogonal to Till’s proposals. That one we need to address
as well (see FLINK-2929). The
There is also this guide:
https://cwiki.apache.org/confluence/display/FLINK/Migration+Guide%3A+0.9.x+to+0.10.x
On Monday, 23 November 2015, Stephan Ewen wrote:
> Hi Arnaud!
>
> In 0.10 , we renamed the dependency to "flink-streaming-java" (and
> flink-streaming-scala"), to be more in line with t
ilian Michels [mailto:m...@apache.org]
> Sent: jeudi 19 novembre 2015 13:36
> To: user@flink.apache.org
> Subject: Re: YARN High Availability
>
> The docs have been updated.
>
> On Thu, Nov 19, 2015 at 12:36 PM, Ufuk Celebi wrote:
>> I’ve added a note about this to the doc
You can use the current release candidate if you like to try it out:
Binaries are here:
http://people.apache.org/~rmetzger/flink-0.10.1-rc1/
The dependency with version 0.10.1 is found in the staging repositories:
https://repository.apache.org/content/repositories/orgapacheflink-1058
If you ca
I’ve added a section about this to the standalone cluster setup guide.
The webpage should be updated tonight by the automatic build bot.
– Ufuk
> On 24 Nov 2015, at 10:39, Welly Tambunan wrote:
>
> Hi Till,
>
> I've just tried that. It's works like a charm. Thanks a lot.
>
> Is there any d
> On 25 Nov 2015, at 02:35, Welly Tambunan wrote:
>
> Hi All,
>
> I would like to know if there any feature differences between using
> Standalone Cluster vs YARN ?
>
> Until now we are using Standalone cluster for our jobs.
> Is there any added value for using YARN ?
>
> We don't have any
> On 01 Dec 2015, at 18:34, Stephan Ewen wrote:
>
> Hi!
>
> If you want to run with checkpoints (fault tolerance), you need to specify a
> place to store the checkpoints to.
>
> By default, it is the master's memory (or zookeeper in HA), so we put a limit
> on the size of the size of the sta
Hey Arnaud,
thanks for reporting this. I think Till’s suggestion will help to debug this
(checking whether a second YARN application has been started)…
You don’t want to run the batch application in HA mode, correct?
I sounds like the batch job is submitted with the same config keys. Could you
> On 03 Dec 2015, at 11:47, LINZ, Arnaud wrote:
>
> Oopss... False joy.
OK, I think this is a bug in the YARN Client and the way it uses the
.properties files to submit jobs.
As a work around: Can you mv the /tmp/.yarn-properties-voyager file and submit
the batch job?
mv /tmp/.yarn-propert
specific execution to have a proper workaround ?
> Thanks,
> Arnaud
>
> -Message d'origine-
> De : Ufuk Celebi [mailto:u...@apache.org]
> Envoyé : jeudi 3 décembre 2015 11:53
> À : user@flink.apache.org
> Objet : Re: HA Mode and standalone containers compatibility ?
Hey Brian,
did you follow the S3 setup guide?
https://ci.apache.org/projects/flink/flink-docs-master/apis/example_connectors.html
You have to set the fs.hdfs.hadoopconf property and add
fs.s3.impl
org.apache.hadoop.fs.s3native.NativeS3FileSystem
to core-site.xml
– Ufuk
> On 09 Dec 2015, at
> On 14 Dec 2015, at 16:25, Niels Basjes wrote:
>
> Hi,
>
> I have a very small topology here.
> In fact this is a thing that generates synthetic data and puts it into Kafka.
> When looking at the web UI I see that all counters (i.e. Bytes received,
> Records received, Bytes sent, Records sen
Hey Brian,
I think that it is currently not possible. I will look into whether there is a
workaround. In any case, this sounds like a useful thing and it shouldn’t be
too complicated to add the desired behaviour.
I’ve opened an issue [1] for it and will look into it tomorrow.
Is this currently
This has been recently added to the YARN client by Robert [1]:
https://ci.apache.org/projects/flink/flink-docs-master/setup/yarn_setup.html#running-flink-on-yarn-behind-firewalls
Are you running YARN?
– Ufuk
[1] https://github.com/apache/flink/pull/1416
> On 15 Dec 2015, at 00:03, Ufuk Cel
On 15 Dec 2015, at 00:28, Cory Monty wrote:
>
> Ufuk,
>
> I'm a colleague of Brian. Unfortunately, we are not running YARN so I don't
> think that PR applies to us. We're trying to run a standalone cluster.
>
> Cheers,
>
> Cory
>
> On Mon, Dec
> On 16 Dec 2015, at 15:00, Ovidiu-Cristian MARCU
> wrote:
>
> Hi
>
> If I restart the Flink I don’t see anymore the history of the completed jobs.
> Is this a missing feature or what should I do to see the completed job list
> history?
Not possible at the moment.
Completed jobs are archive
> On 18 Dec 2015, at 11:07, Philip Lee wrote:
>
> Opps, sorry
>
> I was supposed to email this one to hive mailiing list.
No problem. Can happen easily with auto completion ;)
– Ufuk
> On 17 Dec 2015, at 19:36, Cory Monty wrote:
>
> Hey Ufuk,
>
> We can try building it, but it might be a week or so given the holidays.
> We're currently targeting development against 1.0.0-SNAPSHOT and that release
> is OK unless 1.0 release is more than 2-3 months away.
OK, that’s the exp
@Stephan: It was added to the dependency management section in order to enforce
a higher version for S3 client, because it was causing problems earlier.
> On 06 Jan 2016, at 11:14, Chiwan Park wrote:
>
> Great! Thanks for addressing!
>
>> On Jan 6, 2016, at 5:51 PM, Stephan Ewen wrote:
>>
>>
> On 11 Jan 2016, at 13:03, Madhukar Thota wrote:
>
> Hi
>
> Is streaming supported in Flink-Python API? If so, can you point me to the
> documentation?
No. Only the DataSet API has Python support at the moment. I expect this to
change at some point in time, but I’m not aware of any concret
Hey Welly!
I’m not aware of any concrete plans, but is it possible that you share your
requirements on a high level?
– Ufuk
> On 12 Jan 2016, at 08:24, Welly Tambunan wrote:
>
> Hi Stephen,
>
> Do you have any plan on which encryption method and mechanism will be used on
> Flink ? Could yo
Hey Kovas
sorry for the long delay.
> On 10 Jan 2016, at 06:20, kovas boguta wrote:
> 1) How can I prevent ResultPartitions from being released?
>
> In interactive use, RPs should not necessarily be released when there are no
> pending tasks to consume them.
Max Michels did some work along t
Hey Niels,
as Gabor wrote, this feature has been merged to the master branch recently.
The docs are online here:
https://ci.apache.org/projects/flink/flink-docs-master/apis/savepoints.html
Feel free to report back your experience with it if you give it a try.
– Ufuk
> On 14 Jan 2016, at 11:09
> On 14 Jan 2016, at 22:00, kovas boguta wrote:
>
> On Thu, Jan 14, 2016 at 5:52 AM, Ufuk Celebi wrote:
> Hey Kovas
>
> sorry for the long delay.
>
> It was worth the wait! Thanks for the detailed response.
>
> > Ideally, I could force certain Res
This is now part of the master branch and should be part of the SNAPSHOT builds
soon. The HA docs have a short paragraph on how to configure it.
– Ufuk
> On 21 Dec 2015, at 12:10, Ufuk Celebi wrote:
>
>
>> On 17 Dec 2015, at 19:36, Cory Monty wrote:
>>
>>
> Niels Basjes
>
>
>
>
>
>
> On Mon, Jan 18, 2016 at 12:13 PM, Maximilian Michels wrote:
> The documentation layout changed in the master. Then new URL:
> https://ci.apache.org/projects/flink/flink-docs-master/apis/streaming/savepoints.html
>
> On Thu,
Thanks for the notice. I’ve added a warning to the snapshot docs and created a
Wiki page with the changes:
https://cwiki.apache.org/confluence/display/FLINK/Maven+artifact+names+suffixed+with+Scala+version
– Ufuk
> On 27 Jan 2016, at 12:20, Maximilian Michels wrote:
>
> Dear users and develop
> On 01 Feb 2016, at 17:14, Don Frascuchon wrote:
>
> Hi,
>
> In reference with this topic, there is any feature for automatically restart
> job after a task exception? Like --supervise command line option in apache
> spark
If you are referring to job manager/task manager instances: No. Cur
> On 02 Feb 2016, at 13:28, LINZ, Arnaud wrote:
>
> Thanks,
> I’m using the official 0.10 release. I will try to use the 0.10 snapshot.
>
> FYI, setting the heap cut-off ratio to 0.5 lead to the following error :
That’s the error Stephan was referring to. Does the snapshot version fix it for
> On 02 Feb 2016, at 14:31, LINZ, Arnaud wrote:
>
> Hi,
>
> Unfortunalety, it still fails, but with a different error (see below).
> Note that I did not rebuild & reinstall flink, I just used a 0.10-SNAPSHOT
> compiled jar submitted as a batch job using the "0.10.0" flink installation.
This m
> On 02 Feb 2016, at 15:15, LINZ, Arnaud wrote:
>
> Hi,
>
> Running again with more RAM made the treatement go further, but Yarn still
> killed one container for memory consumption. I will experiment various memory
> parameters.
OK, the killing of the container probably triggered the
Remote
> On 04 Feb 2016, at 12:02, Gwenhael Pasquiers
> wrote:
>
> Ok thanks !
>
> All that’s left is to wait then.
If you have spare time and are working with the current snapshot version, it
would be great to get some feedback on the pull request. :-)
– Ufuk
> On 05 Feb 2016, at 08:56, Jeyhun Karimov wrote:
>
> For example, I will do aggregate operations with other windows (n-window
> aggregations) that are already outputted.
> I tried your suggestion and used filesystem sink, outputted to HDFS.
> I got k files in HDFS directory where k is the num
> On 05 Feb 2016, at 13:28, Paris Carbone wrote:
>
> Hi Gabor,
>
> The sinks should aware that the global checkpoint is indeed persisted before
> emitting so they will have to wait until they are notified for its completion
> before pushing to Kafka. The current view of the local state is not
> On 05 Feb 2016, at 16:38, Flavio Pompermaier wrote:
>
> Is there an easy way to understand if and when my data get skewed in the
> pipeline?
Yes, the web frontend shows how many bytes and records the sub tasks send and
receive respectively. Skew would show as some tasks having higher number
> On 11 Feb 2016, at 22:24, Cory Monty wrote:
>
> Hmm. We don't explicitly include "flink-annotations" and we do not append the
> Scala suffix for "flink-core":
>
> `"org.apache.flink" % "flink-core" % "1.0-SNAPSHOT"`
>
> Here are the packages we currently include with a Scala suffix:
>
> fl
Using the local file system as state backend only works if all job
managers run on the same machine. Is that the case?
Have you specified all job managers in the masters file? With the
local file system state backend only something like
host-X
host-X
host-X
will be a valid masters configuration.
> On 15 Feb 2016, at 13:40, Stefano Baghino
> wrote:
>
> Hi Ufuk, thanks for replying.
>
> Regarding the masters file: yes, I've specified all the masters and checked
> out that they were actually running after the start-cluster.sh. I'll gladly
> share the logs as soon as I get to see them.
I think Chiwan's estimate is accurate. Anything between 2 and 4 weeks
is realistic in my opinion. We will make sure that the release comes
with a migration/breaking changes guide, so you will have a smooth
experience when upgrading.
In the mean time, you can also work with the current master/snaps
Hey Zach!
Sounds like a great use case.
On Wed, Feb 17, 2016 at 3:16 PM, Zach Cox wrote:
> However, the savepoint docs state that the job parallelism cannot be changed
> over time [1]. Does this mean we need to use the same, fixed parallelism=n
> during reprocessing and going forward? Are there
On Thu, Feb 18, 2016 at 6:59 PM, Thomas Lamirault
wrote:
> We are trying flink in HA mode.
Great to hear!
> We set in the flink yaml :
>
> state.backend: filesystem
>
> recovery.mode: zookeeper
> recovery.zookeeper.quorum:
>
> recovery.zookeeper.path.root:
>
> recovery.zookeeper.storageDir:
>
On Sat, Feb 20, 2016 at 10:12 AM, Aljoscha Krettek wrote:
> IMHO the only change for 2) is that you possibly get better machine
> utilization because it will use more parallel threads. So I think it’s a
> valid approach.
>
> @Ufuk, could there be problems with the number of network buffers? I t
Note that the method to call in the example should be
`conf.setInteger` and the second argument not a String but an int.
On Sun, Feb 21, 2016 at 1:41 PM, Márton Balassi
wrote:
> Dear Ana,
>
> If you are using a single machine with multiple cores, but need convenient
> access to the configuration
The new default is equivalent to the previous "streaming mode". The
community decided to get rid of this distinction, because it was
confusing to users.
The difference between "streaming mode" and "batch mode" was how
Flink's managed memory was allocated, either lazily when required
('streaming mo
On Tue, Feb 23, 2016 at 10:17 AM, Ana M. Martinez wrote:
> I believe that setting taskmanager.numberOfTaskSlots is not necessary, but
> setParallelism is, as by default 1 was taken.
Yes, the number of slots in local execution defaults to the maximum
parallelism of the job.
ny trade-off that we should
> know etc ?
>
> Cheers
>
> On Tue, Feb 23, 2016 at 3:03 PM, Welly Tambunan wrote:
>>
>> Hi Ufuk,
>>
>> Thanks for the explanation.
>>
>> Yes. Our jobs is all streaming job.
>>
>> Cheers
>>
>> On
Hey Zach! I'm not aware of an open issue for this.
You can go ahead and open an issue for it. It will be very helpful to
include the following:
- exact Chrome and OS X version
- the exectuion plan as JSON (via env.getExecutionPlan())
- screenshot
Thanks!
– Ufuk
On Tue, Feb 23, 2016 at 3:46 PM,
Thanks! :-)
I hope we can fix it for the release.
On Tue, Feb 23, 2016 at 4:45 PM, Zach Cox wrote:
> Hi Ufuk - here is the jira issue with the requested information:
> https://issues.apache.org/jira/browse/FLINK-3483
>
> -Zach
>
>
> On Tue, Feb 23, 2016 at 8:59 AM Ufuk Cel
You can use the environment to set it the job parallelism to 6 e.g.
env.setParallelism(6).
Setting this will override the default behaviour. Maybe that's why the
default parallelism is not working... you might have it set to 1
already?
On Wed, Feb 24, 2016 at 3:41 PM, Saiph Kappa wrote:
> I set
Hey Chris!
I think that the full amount of memory to Flink leads to the TM
process being killed by the OS. Can you check the OS logs whether the
OOM killer shut it down? You should be able to see this in the system
logs.
– Ufuk
On Thu, Feb 25, 2016 at 11:24 AM, Boden, Christoph
wrote:
> Dear F
Hey Welly!
Yes, it is possible to do manually via the jobmanager.sh and
taskmanager.sh script like this:
jobmanager.sh start cluster $HOST $WEB-UI-PORT
taskmanager.sh start
The start-cluster.sh script is just a wrapper around these scripts.
>From experience, it's often forgotten to sync the con
Hey Max!
for the first WARN in
org.apache.flink.runtime.webmonitor.JobManagerRetriever: this is
expected if the new leader has not updated ZooKeeper yet. The
important thing is that the new leading job manager is eventually
retrieved. This did happen, right?
Regarding eth1 vs. eth0: After the new
). This is the only workaroud I'm aware of
at this point. Maybe Till can chime in here whether this has other
implications as well?
– Ufuk
On Thu, Mar 3, 2016 at 9:59 AM, Ufuk Celebi wrote:
> Hey Max!
>
> for the first WARN in
> org.apache.flink.runtime.webmonitor.JobManagerR
Hey Vijay!
On Mon, Mar 7, 2016 at 8:42 PM, Vijay Srinivasaraghavan
wrote:
> 3) How can I simulate and verify backpressure? I have introduced some delay
> (Thread Sleep) in the job before the sink but the "backpressure" tab from UI
> does not show any indication of whether backpressure is working
I've added this to the migration guide here:
https://cwiki.apache.org/confluence/display/FLINK/Migration+Guide%3A+0.10.x+to+1.0.x
Feel free to add any other API changes that are missing there.
– Ufuk
On Thu, Mar 10, 2016 at 10:13 AM, Aljoscha Krettek wrote:
> Hi,
> you’re right, this should be
Just talked with Stephan: the document you are referring to is stale.
Can you check out this one here:
https://ci.apache.org/projects/flink/flink-docs-release-1.0/apis/streaming/event_time.html
– Ufuk
On Thu, Mar 10, 2016 at 10:17 AM, Ufuk Celebi wrote:
> I've added this to the migrati
Hey Maciek! I'm working on the other proposed fix by closing the
buffer pool early. I expect the fix to make it into the next bugfix
release 1.0.1 (or 1.0.2 if 1.0.1 comes very soon).
– Ufuk
Just removed the page. Triggering a new docs build...
On Thu, Mar 10, 2016 at 10:22 AM, Aljoscha Krettek wrote:
> Then Stephan should have removed the old doc when adding the new one… :-)
>> On 10 Mar 2016, at 10:20, Ufuk Celebi wrote:
>>
>> Just talked with Stephan:
the backpressure tab from UI to display some warning but
> I still see "OK" message.
>
> This makes me wonder if I am testing the backpressure scenario properly or
> not?
>
> Regards
> Vijay
>
> On Monday, March 7, 2016 3:19 PM, Ufuk Celebi wrote:
>
>
Hey Ravinder,
check out the following config keys:
blob.server.port
taskmanager.rpc.port
taskmanager.data.port
– Ufuk
On Wed, Feb 10, 2016 at 4:06 PM, Ravinder Kaur wrote:
> Hello Fabian,
>
> Thank you very much for the resource. I had already gone through this and
> have found port '6123' a
I was wondering whether we should completely remove that page and just
link to the examples package on GitHub. Do you think that it would be
a good idea?
On Fri, Mar 11, 2016 at 10:45 AM, Maximilian Michels wrote:
> Thanks for noticing, Janardhan. Fixed for the next release.
>
> On Fri, Mar 11, 2
Hey Nick!
I just checked and the conf/log4j.properties file is copied and is
given as an argument to the JVM.
You should see the following:
- client logs that the conf/log4j.properties file is copied
- JobManager logs show log4j.configuration being passed to the JVM.
Can you confirm that these s
Everything in the lib folder should be added to the classpath. Can you
check the YARN client logs that the files are uploaded? Furthermore,
you can check the classpath of the JVM in the YARN logs of the
JobManager/TaskManager processes.
– Ufuk
On Fri, Mar 11, 2016 at 5:33 PM, Andrea Sella
wrote
Probably the limitation is that the number of keys is different in the
real and the synthetic data set respectively. Can you confirm this?
The solution set for delta iterations is currently implemented as an
in-memory hash table that works on managed memory segments, but is not
spillable.
– Ufuk
What do you mean with iteration in this context? Are you repeatedly
running the same WordCount program for streaming and batch
respectively?
– Ufuk
On Tue, Mar 15, 2016 at 10:22 AM, Till Rohrmann wrote:
> Hi Ravinder,
>
> could you tell us what's written in the taskmanager log of the failing
> t
to specify -t lib.
>
> If you can double check the issue I can open an issue on JIRA.
>
> Thanks for helping us.
>
> On Thu, Mar 17, 2016 at 2:16 PM, Ufuk Celebi wrote:
>>
>> Can you try the same thing without -yt, but a yarn-session?
>>
>> – Ufuk
>&g
On Thu, Mar 17, 2016 at 11:51 AM, Stefano Baghino
wrote:
> does the recovery.zookeeper.path.root property need to be set independently
> for each job that is run?
No, just per cluster.
Are you facing these issues with the batch or streaming programs?
– Ufuk
On Wed, Mar 16, 2016 at 4:30 PM, Till Rohrmann wrote:
> If the problem is that your JVMs stall too long, then you can also increase
> the akka.ask.timeout configuration value in flink-config.yaml. That will
> also increase
t;
>>> On Sat, Mar 12, 2016 at 3:25 PM, Andrea Sella
>>> wrote:
>>>>
>>>> Hi Ufuk,
>>>>
>>>> I'm trying to execute the WordCount batch example with input and output
>>>> on Alluxio, i followed Running Flink on Allu
Hey Souringa,
could you provide some context about the program you are running?
Is it batch or streaming? What is the parallelism? How many operators are
you running?
Thanks for reporting the issue. I think we will figure it out once you
provide further information. :-)
– Ufuk
On Fri, Mar 18,
erGroupInformation.(UserGroupInformation.java:84)
> ... 3 more
> End of LogType:jobmanager.err
>
> But if I just add -yt lib (the Flink lib folder already contains SLF4J and
> Log4J) the libraries are shipped and the job goes on.
>
>
> On Wed, Mar 16, 2016 at 9:15 PM, Ufuk Cel
eckpoints are meaningful or not.
>
>
>
> 2016-03-16 15:33 GMT+01:00 Ufuk Celebi :
>>
>> Can you please have a look into the JobManager log file and report
>> which checkpoints are restored? You should see messages from
>> ZooKeeperCompletedCheckpointStore like:
>> - Fou
tion is not
>> properly configured so I wouldn't rely too much on that. Before doing the
>> tests tomorrow I will clear all the existing logs just to be sure.
>>
>> 2016-03-16 18:19 GMT+01:00 Ufuk Celebi :
>>>
>>> OK, so you are submitting multiple jobs
on -kill left the "flink run" process alive so that
> may be the problem. We just noticed a few minutes ago.
>
> If the problem persists, I will eventually come back with a full log.
>
> Thanks for now,
>
> Simone
>
> 2016-03-16 18:04 GMT+01:00 Ufuk Celebi :
>>
&
tly
> for each job that is run? Doesn't Flink take care of assigning some sort of
> identification to each job and storing their checkpoints independently?
>
> On Thu, Mar 17, 2016 at 11:43 AM, Ufuk Celebi wrote:
>>
>> Hey Simone! Did you set different recovery.zooke
Can you please have a look into the JobManager log file and report
which checkpoints are restored? You should see messages from
ZooKeeperCompletedCheckpointStore like:
- Found X checkpoints in ZooKeeper
- Initialized with X. Removing all older checkpoints
You can share the complete job manager log
Hey Gwenhaël,
see here for recursive traversal of input paths:
https://ci.apache.org/projects/flink/flink-docs-master/apis/batch/index.html#recursive-traversal-of-the-input-path-directory
Regarding the phases: the best way to exchange data between batch jobs
is via files. You can then execute two
If you want all sub directories under data/2016/01, then this could help:
https://ci.apache.org/projects/flink/flink-docs-master/apis/batch/index.html#recursive-traversal-of-the-input-path-directory
On Mon, Mar 21, 2016 at 11:35 AM, Fabian Hueske wrote:
> Hi,
>
> no, this is currently not suppor
Hey Ravinder,
can you please share the JobManager logs as well?
The logs say that the TaskManager disconnects from the JobManager,
because that one is not reachable anymore. At this point, the running
shuffles are cancelled and you see the follow up
RemoteTransportExceptions.
– Ufuk
On Mon, Ma
Nice! Would you like to contribute this to Flink via a pull request? Some
resources about the contribution process can be found here:
http://flink.apache.org/contribute-code.html
http://flink.apache.org/how-to-contribute.html
On Wed, Mar 23, 2016 at 12:00 AM, Fabian Hueske wrote:
> Hi Gna,
>
>
Hey Sourigna,
that particular method is not part of Flink yet.
Did you have a look at the sampling methods in DataSetUtils? Maybe they can
be helpful for what you are trying to achieve.
– Ufuk
On Wed, Mar 23, 2016 at 5:19 PM, Sourigna Phetsarath <
gna.phetsar...@teamaol.com> wrote:
> All:
>
>
Hey!
(a) This is not a Flink term. I could not find the term in the slides,
but I guess that it is referring to tumbling windows. For more
details, check out these pages:
https://flink.apache.org/news/2015/12/04/Introducing-windows.html
https://ci.apache.org/projects/flink/flink-docs-release-1.0
sampling methods in DataSetUtils and they are
> helpful.
>
> Just wanted to see if that particular method is on the road map for a
> future release.
>
> -Gna
>
> On Mon, Mar 28, 2016 at 6:22 AM, Ufuk Celebi wrote:
>
>> Hey Sourigna,
>>
>> that particu
Hey Stefano,
this should work by setting the parallelism on the environment, e.g.
env.setParallelism(32)
Is this what you are doing?
The task threads are not part of a pool, but each submitted task
creates its own Thread.
– Ufuk
On Fri, Mar 25, 2016 at 9:10 PM, Flavio Pompermaier
wrote:
> A
ctual starting of the threads, the number is
>>>>>> fix to 8. We run a debugger to get to the point where the thread was
>>>>>> started. As Flavio mentioned, the ExecutionContext has the parallelims
>>>>>> set
>>>>>&
one job. That way we are sure to isolate
>> the different jobs as much as possible and in case of crashes / bugs /
>> (etc) can completely kill one cluster without interfering with the other
>> jobs.
>>
>> That future behavior seems good :-)
>>
>> Instead of
Hey Stefano,
yarn.resourcemanager.am.max-attempts is a setting for your YARN
cluster and cannot be influenced by Flink. Flink cannot set a higher
number than this for yarn.application-attempts.
The key that is set/overriden by Flink is probably only valid for the
YARN session, but I'm not too fam
Same here.
+1 to publish
On Mon, Apr 4, 2016 at 10:04 AM, Aljoscha Krettek wrote:
> Hi,
> I like it. Very dense and very focused on the example but I think it should
> be good for the Flink Blog.
>
> --
> aljoscha
>
> On Fri, 1 Apr 2016 at 15:56 Till Rohrmann wrote:
>
>> Hi Flink community,
>>
Just to clarify: Shinhyung is running one a single node with 4 CPUs,
each having 16 cores.
On Mon, Apr 4, 2016 at 10:32 AM, Robert Metzger wrote:
> Hi,
>
> usually it doesn't make sense to run multiple task managers on a single
> machine to get more slots.
> Your machine has only 4 CPU cores, so
Dear Flink community,
I have updated the Material section on the Flink project page and
moved the slides section to a separate page.
You can find links to slides and talks here now:
http://flink.apache.org/slides.html
I've added slides for talks from this year by Till Rohrmann, Vasia
Kalavri, Ro
Dear Flink community,
I'm happy to announce that we have added a long overdue section on
general Flink concepts to the documentation.
You can find it here:
https://ci.apache.org/projects/flink/flink-docs-release-1.0/concepts/concepts.html
Thanks to Stephan Ewen who wrote this great overview. I h
@Paris: Just added it. Thanks for the pointer. Great slides!
lient requested shutdown.
>>
>> The YarnJobManager starts up properly, but the Client never sends
>> anything, shuts down at some point, and tears down the YARN cluster.
>>
>> Client logs would help a lot there...
>>
>>
>>
>>
>> On Sat, Ap
1 - 100 of 546 matches
Mail list logo