I think that the machines have lost connection. That is most likely
connected to the heartbeat interval of the watch or transport failure
detector. The transport failure detector should actually be set to a
heartbeat interval of 1000 s and consequently it should not cause any
problems.

Which version of the snapshot are you exactly using Christoph? If there is
still the config parameter akka.loglevel could you please set it to INFO
and send us the log output again. If you use the latest snapshot, then this
should no longer be necessary. Alternatively, you could try to set the
akka.watch.heartbeat.interval and akka.watch.heartbeat.pause to something
like 1000 s and 6000 s, respectively, and try it again.

Are you running the experiments on cloud-11 Christoph? Maybe I can take a
look at it if you can provide me the job jars and the Flink version.

On Wed, Jan 28, 2015 at 7:57 PM, Stephan Ewen <se...@apache.org> wrote:

> I see the following line:
>
> 11:14:32,603 WARN  akka.remote.ReliableDeliverySupervisor
>       - Association with remote system [akka.tcp://
> fl...@cloud-26.dima.tu-berlin.de:51449] has failed, address is now gated
> for [5000] ms. Reason is: [Disassociated].
>
> Does that mean that the machines have lost connection?
>
> @Till What is your take on this?
>
>
> On Wed, Jan 28, 2015 at 7:07 AM, Bruecke, Christoph <
> christoph.brue...@campus.tu-berlin.de> wrote:
>
> > I might add that the error only occurs when running with the
> > RemoteExecutor regardless of the number of TM. Starting the job in
> IntelliJ
> > with the LocalExecutor with dop 1 works just fine.
> >
> > Best,
> > Christoph
> >
> > On 28 Jan 2015, at 12:17, Bruecke, Christoph <
> > christoph.brue...@campus.tu-berlin.de> wrote:
> >
> > > Hi Robert,
> > >
> > > thanks for the quick response. Here is the jobmanager-main.log:
> > >
> > > PS: I’m subscribed now.
> > >
> > > 11:09:16,144 INFO  org.apache.flink.yarn.ApplicationMaster$
> >         - YARN daemon runs as hadoop setting user to execute Flink
> > ApplicationMaster/JobManager to hadoop
> > > 11:09:16,199 INFO  org.apache.flink.yarn.Utils
> >          - Found YARN_CONF_DIR, adding it to configuration
> > > 11:09:16,298 INFO  org.apache.flink.yarn.ApplicationMaster$
> >         - Start job manager for yarn
> > > 11:09:16,298 INFO  org.apache.flink.yarn.ApplicationMaster$
> >         - Config path:
> >
> /data/3/hadoop-2.4/yarn-local-dir/usercache/hadoop/appcache/application_1420727594991_0068/container_1420727594991_0068_01_000001.
> > > 11:09:17,179 INFO  akka.event.slf4j.Slf4jLogger
> >         - Slf4jLogger started
> > > 11:09:17,271 INFO  Remoting
> >         - Starting remoting
> > > 11:09:17,556 INFO  Remoting
> >         - Remoting started; listening on addresses :[akka.tcp://
> > fl...@cloud-29.dima.tu-berlin.de:42643]
> > > 11:09:17,569 INFO  org.apache.flink.yarn.ApplicationMaster$
> >         - Start job manager actor.
> > > 11:09:17,570 INFO  org.apache.flink.yarn.ApplicationMaster$
> >         - Generate configuration file for application master.
> > > 11:09:17,579 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Starting job manager at akka://flink/user/jobmanager.
> > > 11:09:17,590 INFO  org.apache.flink.yarn.ApplicationMaster$
> >         - Start yarn session on job manager.
> > > 11:09:17,591 INFO  org.apache.flink.yarn.ApplicationMaster$
> >         - Application Master properly initiated. Await termination of
> actor
> > system.
> > > 11:09:17,601 INFO  org.apache.flink.runtime.blob.BlobServer
> >         - Started BLOB server on port 35605
> > > 11:09:17,616 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Started job manager. Waiting for incoming messages.
> > > 11:09:17,621 INFO
> > org.apache.flink.runtime.jobmanager.web.WebInfoServer         - Setting
> up
> > web info server, using web-root
> >
> directoryjar:file:/data/4/hadoop-2.4/yarn-local-dir/usercache/hadoop/appcache/application_1420727594991_0068/filecache/11/flink-dist-0.9-SNAPSHOT-yarn-uberjar.jar!/web-docs-infoserver.
> > > 11:09:17,621 INFO
> > org.apache.flink.runtime.jobmanager.web.WebInfoServer         - Web info
> > server will display information about flink job-manager on localhost,
> port
> > 8081.
> > > 11:09:17,733 INFO
> > org.apache.flink.runtime.jobmanager.web.WebInfoServer         - Starting
> > web info server for JobManager on port 8081
> > > 11:09:17,734 INFO  org.eclipse.jetty.util.log
> >         - jetty-0.9-SNAPSHOT
> > > 11:09:17,770 INFO  org.eclipse.jetty.util.log
> >         - Started SelectChannelConnector@0.0.0.0:8081
> > > 11:09:17,784 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Start yarn session.
> > > 11:09:17,784 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Requesting 2 task managers.
> > > 11:09:18,414 INFO  org.apache.hadoop.yarn.client.RMProxy
> >          - Connecting to ResourceManager at cloud-11/130.149.21.15:8030
> > > 11:09:18,423 INFO
> > org.apache.hadoop.yarn.client.api.impl.ContainerManagementProtocolProxy
> -
> > yarn.client.max-nodemanagers-proxies : 500
> > > 11:09:18,425 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Registering ApplicationMaster with tracking url
> > http://cloud-29.dima.tu-berlin.de:8081.
> > > 11:09:18,787 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Requesting TaskManager container 0.
> > > 11:09:18,794 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Requesting TaskManager container 1.
> > > 11:09:18,903 INFO  org.apache.flink.yarn.Utils
> >          - Copying from
> >
> file:/data/3/hadoop-2.4/yarn-local-dir/usercache/hadoop/appcache/application_1420727594991_0068/container_1420727594991_0068_01_000001/flink-conf-modified.yaml
> > to hdfs://
> >
> cloud-11.dima.tu-berlin.de:60010/user/hadoop/.flink/application_1420727594991_0068/flink-conf-modified.yaml
> > > 11:09:19,280 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Prepared local resource for modified yaml: resource { scheme: "hdfs"
> > host: "cloud-11.dima.tu-berlin.de" port: 60010 file:
> >
> "/user/hadoop/.flink/application_1420727594991_0068/flink-conf-modified.yaml"
> > } size: 3121 timestamp: 1422439759181 type: FILE visibility: APPLICATION
> > > 11:09:19,286 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Create container launch context.
> > > 11:09:19,300 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Starting TM with command=$JAVA_HOME/bin/java -Xmx819m
> > -Dlog.file="<LOG_DIR>/taskmanager.log"
> > -Dlogback.configurationFile=file:logback.xml
> > -Dlog4j.configuration=file:log4j.properties
> > org.apache.flink.yarn.appMaster.YarnTaskManagerRunner --configDir . 1>
> > <LOG_DIR>/taskmanager-stdout.log 2> <LOG_DIR>/taskmanager-stderr.log
> > > 11:09:19,619 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Got new container for TM container_1420727594991_0068_01_000002 on host
> > cloud-26.dima.tu-berlin.de
> > > 11:09:19,621 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Launching container #1.
> > > 11:09:19,622 INFO
> > org.apache.hadoop.yarn.client.api.impl.ContainerManagementProtocolProxy
> -
> > Opening proxy : cloud-26.dima.tu-berlin.de:8045
> > > 11:09:19,828 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Got new container for TM container_1420727594991_0068_01_000003 on host
> > cloud-31.dima.tu-berlin.de
> > > 11:09:19,829 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Launching container #2.
> > > 11:09:19,831 INFO
> > org.apache.hadoop.yarn.client.api.impl.ContainerManagementProtocolProxy
> -
> > Opening proxy : cloud-31.dima.tu-berlin.de:8045
> > > 11:09:25,748 INFO  org.apache.flink.runtime.instance.InstanceManager
> >          - Registered TaskManager at akka.tcp://
> > fl...@cloud-26.dima.tu-berlin.de:51449/user/taskmanager as
> > ca731dbbfe941a63077c9c148254607c. Current number of registered hosts is
> 1.
> > > 11:09:25,845 INFO  org.apache.flink.runtime.instance.InstanceManager
> >          - Registered TaskManager at akka.tcp://
> > fl...@cloud-31.dima.tu-berlin.de:43200/user/taskmanager as
> > 7bf3aa638e00682239c9d8460d142191. Current number of registered hosts is
> 2.
> > > 11:11:13,837 WARN  akka.remote.ReliableDeliverySupervisor
> >         - Association with remote system [akka.tcp://
> > CliFrontendActorSystem@130.149.21.15:50292] has failed, address is now
> > gated for [5000] ms. Reason is: [Disassociated].
> > > 11:14:30,892 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Received job c979ea0b2e358fb988f5a2a30ab48786 (Flink Java Job at Wed
> Jan
> > 28 11:14:28 CET 2015).
> > > 11:14:30,997 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - job.end.retry.interval is deprecated. Instead, use
> > mapreduce.job.end-notification.retry.interval
> > > 11:14:30,998 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - io.bytes.per.checksum is deprecated. Instead, use
> > dfs.bytes-per-checksum
> > > 11:14:30,999 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker.retiredjobs.cache.size is deprecated.
> Instead,
> > use mapreduce.jobtracker.retiredjobs.cache.size
> > > 11:14:31,000 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.profile.reduces is deprecated. Instead, use
> > mapreduce.task.profile.reduces
> > > 11:14:31,000 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.reuse.jvm.num.tasks is deprecated. Instead, use
> > mapreduce.job.jvm.numtasks
> > > 11:14:31,001 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.reduce.tasks.speculative.execution is deprecated.
> Instead,
> > use mapreduce.reduce.speculative
> > > 11:14:31,003 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.tracker.http.address is deprecated. Instead, use
> > mapreduce.tasktracker.http.address
> > > 11:14:31,004 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.system.dir is deprecated. Instead, use
> > mapreduce.jobtracker.system.dir
> > > 11:14:31,005 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.tracker.report.address is deprecated. Instead, use
> > mapreduce.tasktracker.report.address
> > > 11:14:31,006 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.healthChecker.interval is deprecated. Instead, use
> > mapreduce.tasktracker.healthchecker.interval
> > > 11:14:31,007 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.child.tmp is deprecated. Instead, use
> > mapreduce.task.tmp.dir
> > > 11:14:31,008 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.taskmemorymanager.monitoring-interval is
> > deprecated. Instead, use
> > mapreduce.tasktracker.taskmemorymanager.monitoringinterval
> > > 11:14:31,008 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.shuffle.connect.timeout is deprecated. Instead, use
> > mapreduce.reduce.shuffle.connect.timeout
> > > 11:14:31,008 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.speculative.execution.speculativeCap is deprecated.
> > Instead, use mapreduce.job.speculative.speculativecap
> > > 11:14:31,009 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - io.sort.spill.percent is deprecated. Instead, use
> > mapreduce.map.sort.spill.percent
> > > 11:14:31,010 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.shuffle.input.buffer.percent is deprecated. Instead,
> > use mapreduce.reduce.shuffle.input.buffer.percent
> > > 11:14:31,010 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.skip.map.max.skip.records is deprecated. Instead, use
> > mapreduce.map.skip.maxrecords
> > > 11:14:31,011 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.profile.maps is deprecated. Instead, use
> > mapreduce.task.profile.maps
> > > 11:14:31,012 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.local.dir is deprecated. Instead, use
> > mapreduce.cluster.local.dir
> > > 11:14:31,013 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.merge.recordsBeforeProgress is deprecated. Instead, use
> > mapreduce.task.merge.progress.records
> > > 11:14:31,013 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker.http.address is deprecated. Instead, use
> > mapreduce.jobtracker.http.address
> > > 11:14:31,013 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.userlog.retain.hours is deprecated. Instead, use
> > mapreduce.job.userlog.retain.hours
> > > 11:14:31,014 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.compress.map.output is deprecated. Instead, use
> > mapreduce.map.output.compress
> > > 11:14:31,014 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.speculative.execution.slowNodeThreshold is deprecated.
> > Instead, use mapreduce.job.speculative.slownodethreshold
> > > 11:14:31,015 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.reduce.tasks.maximum is deprecated. Instead,
> > use mapreduce.tasktracker.reduce.tasks.maximum
> > > 11:14:31,017 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.reduce.child.log.level is deprecated. Instead, use
> > mapreduce.reduce.log.level
> > > 11:14:31,018 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.jobtracker.restart.recover is deprecated. Instead, use
> > mapreduce.jobtracker.restart.recover
> > > 11:14:31,020 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.inmem.merge.threshold is deprecated. Instead, use
> > mapreduce.reduce.merge.inmem.threshold
> > > 11:14:31,021 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.reduce.tasks is deprecated. Instead, use
> > mapreduce.job.reduces
> > > 11:14:31,023 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.acls.enabled is deprecated. Instead, use
> > mapreduce.cluster.acls.enabled
> > > 11:14:31,023 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.dns.nameserver is deprecated. Instead, use
> > mapreduce.tasktracker.dns.nameserver
> > > 11:14:31,024 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.submit.replication is deprecated. Instead, use
> > mapreduce.client.submit.file.replication
> > > 11:14:31,024 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.map.tasks.speculative.execution is deprecated. Instead,
> > use mapreduce.map.speculative
> > > 11:14:31,025 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.map.child.log.level is deprecated. Instead, use
> > mapreduce.map.log.level
> > > 11:14:31,026 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.map.max.attempts is deprecated. Instead, use
> > mapreduce.map.maxattempts
> > > 11:14:31,026 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.shuffle.merge.percent is deprecated. Instead, use
> > mapreduce.reduce.shuffle.merge.percent
> > > 11:14:31,026 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker.jobhistory.lru.cache.size is deprecated.
> > Instead, use mapreduce.jobtracker.jobhistory.lru.cache.size
> > > 11:14:31,029 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.queue.name is deprecated. Instead, use
> > mapreduce.job.queuename
> > > 11:14:31,030 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker.persist.jobstatus.hours is deprecated.
> > Instead, use mapreduce.jobtracker.persist.jobstatus.hours
> > > 11:14:31,031 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.healthChecker.script.timeout is deprecated. Instead, use
> > mapreduce.tasktracker.healthchecker.script.timeout
> > > 11:14:31,031 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.map.tasks.maximum is deprecated. Instead,
> use
> > mapreduce.tasktracker.map.tasks.maximum
> > > 11:14:31,032 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.reduce.markreset.buffer.percent is deprecated.
> > Instead, use mapreduce.reduce.markreset.buffer.percent
> > > 11:14:31,032 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - jobclient.completion.poll.interval is deprecated. Instead, use
> > mapreduce.client.completion.pollinterval
> > > 11:14:31,032 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker.persist.jobstatus.dir is deprecated.
> Instead,
> > use mapreduce.jobtracker.persist.jobstatus.dir
> > > 11:14:31,033 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.reduce.slowstart.completed.maps is deprecated. Instead,
> > use mapreduce.job.reduce.slowstart.completedmaps
> > > 11:14:31,034 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - io.sort.mb is deprecated. Instead, use
> mapreduce.task.io.sort.mb
> > > 11:14:31,034 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - dfs.umaskmode is deprecated. Instead, use
> > fs.permissions.umask-mode
> > > 11:14:31,037 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.jobtracker.instrumentation is deprecated. Instead, use
> > mapreduce.jobtracker.instrumentation
> > > 11:14:31,038 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - topology.node.switch.mapping.impl is deprecated. Instead, use
> > net.topology.node.switch.mapping.impl
> > > 11:14:31,039 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.output.compression.type is deprecated. Instead, use
> > mapreduce.output.fileoutputformat.compress.type
> > > 11:14:31,039 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.skip.attempts.to.start.skipping is deprecated. Instead,
> > use mapreduce.task.skip.start.attempts
> > > 11:14:31,040 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.tracker.task-controller is deprecated. Instead, use
> > mapreduce.tasktracker.taskcontroller
> > > 11:14:31,041 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.userlog.limit.kb is deprecated. Instead, use
> > mapreduce.task.userlog.limit.kb
> > > 11:14:31,043 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.committer.job.setup.cleanup.needed is deprecated.
> Instead,
> > use mapreduce.job.committer.setup.cleanup.needed
> > > 11:14:31,043 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.map.tasks is deprecated. Instead, use mapreduce.job.maps
> > > 11:14:31,044 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.local.dir.minspacekill is deprecated. Instead, use
> > mapreduce.tasktracker.local.dir.minspacekill
> > > 11:14:31,044 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapreduce.jobtracker.split.metainfo.maxsize is deprecated.
> > Instead, use mapreduce.job.split.metainfo.maxsize
> > > 11:14:31,045 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - jobclient.progress.monitor.poll.interval is deprecated.
> Instead,
> > use mapreduce.client.progressmonitor.pollinterval
> > > 11:14:31,046 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.min.split.size is deprecated. Instead, use
> > mapreduce.input.fileinputformat.split.minsize
> > > 11:14:31,047 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.output.compression.codec is deprecated. Instead, use
> > mapreduce.output.fileoutputformat.compress.codec
> > > 11:14:31,048 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.profile is deprecated. Instead, use
> > mapreduce.task.profile
> > > 11:14:31,048 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.reduce.parallel.copies is deprecated. Instead, use
> > mapreduce.reduce.shuffle.parallelcopies
> > > 11:14:31,049 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - io.sort.factor is deprecated. Instead, use
> > mapreduce.task.io.sort.factor
> > > 11:14:31,050 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.input.dir is deprecated. Instead, use
> > mapreduce.input.fileinputformat.inputdir
> > > 11:14:31,050 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.timeout is deprecated. Instead, use
> > mapreduce.task.timeout
> > > 11:14:31,052 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.heartbeats.in.second is deprecated. Instead, use
> > mapreduce.jobtracker.heartbeats.in.second
> > > 11:14:31,053 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.task.cache.levels is deprecated. Instead, use
> > mapreduce.jobtracker.taskcache.levels
> > > 11:14:31,053 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.map.output.compression.codec is deprecated. Instead, use
> > mapreduce.map.output.compress.codec
> > > 11:14:31,054 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.dns.interface is deprecated. Instead, use
> > mapreduce.tasktracker.dns.interface
> > > 11:14:31,055 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.max.tracker.failures is deprecated. Instead, use
> > mapreduce.job.maxtaskfailures.per.tracker
> > > 11:14:31,056 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - dfs.df.interval is deprecated. Instead, use fs.df.interval
> > > 11:14:31,056 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.tasks.sleeptime-before-sigkill is
> deprecated.
> > Instead, use mapreduce.tasktracker.tasks.sleeptimebeforesigkill
> > > 11:14:31,057 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.max.tracker.blacklists is deprecated. Instead, use
> > mapreduce.jobtracker.tasktracker.maxblacklists
> > > 11:14:31,058 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - jobclient.output.filter is deprecated. Instead, use
> > mapreduce.client.output.filter
> > > 11:14:31,059 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - job.end.retry.attempts is deprecated. Instead, use
> > mapreduce.job.end-notification.retry.attempts
> > > 11:14:31,059 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.jobtracker.taskScheduler is deprecated. Instead, use
> > mapreduce.jobtracker.taskscheduler
> > > 11:14:31,060 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.speculative.execution.slowTaskThreshold is deprecated.
> > Instead, use mapreduce.job.speculative.slowtaskthreshold
> > > 11:14:31,061 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.indexcache.mb is deprecated. Instead, use
> > mapreduce.tasktracker.indexcache.mb
> > > 11:14:31,061 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - tasktracker.http.threads is deprecated. Instead, use
> > mapreduce.tasktracker.http.threads
> > > 11:14:31,062 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker.handler.count is deprecated. Instead, use
> > mapreduce.jobtracker.handler.count
> > > 11:14:31,062 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - keep.failed.task.files is deprecated. Instead, use
> > mapreduce.task.files.preserve.failedtasks
> > > 11:14:31,062 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.output.compress is deprecated. Instead, use
> > mapreduce.output.fileoutputformat.compress
> > > 11:14:31,063 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.jobtracker.job.history.block.size is deprecated.
> Instead,
> > use mapreduce.jobtracker.jobhistory.block.size
> > > 11:14:31,063 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.skip.reduce.max.skip.groups is deprecated. Instead, use
> > mapreduce.reduce.skip.maxgroups
> > > 11:14:31,066 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - topology.script.number.args is deprecated. Instead, use
> > net.topology.script.number.args
> > > 11:14:31,066 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - fs.default.name is deprecated. Instead, use fs.defaultFS
> > > 11:14:31,067 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.local.dir.minspacestart is deprecated. Instead, use
> > mapreduce.tasktracker.local.dir.minspacestart
> > > 11:14:31,067 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.jobtracker.maxtasks.per.job is deprecated. Instead, use
> > mapreduce.jobtracker.maxtasks.perjob
> > > 11:14:31,068 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.reduce.max.attempts is deprecated. Instead, use
> > mapreduce.reduce.maxattempts
> > > 11:14:31,068 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker is deprecated. Instead, use
> > mapreduce.jobtracker.address
> > > 11:14:31,069 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.shuffle.read.timeout is deprecated. Instead, use
> > mapreduce.reduce.shuffle.read.timeout
> > > 11:14:31,069 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.instrumentation is deprecated. Instead, use
> > mapreduce.tasktracker.instrumentation
> > > 11:14:31,070 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.tasktracker.expiry.interval is deprecated. Instead, use
> > mapreduce.jobtracker.expire.trackers.interval
> > > 11:14:31,071 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.tracker.persist.jobstatus.active is deprecated.
> > Instead, use mapreduce.jobtracker.persist.jobstatus.active
> > > 11:14:31,074 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.temp.dir is deprecated. Instead, use
> > mapreduce.cluster.temp.dir
> > > 11:14:31,075 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - hadoop.native.lib is deprecated. Instead, use
> > io.native.lib.available
> > > 11:14:31,075 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapreduce.job.counters.limit is deprecated. Instead, use
> > mapreduce.job.counters.max
> > > 11:14:31,076 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapred.job.reduce.input.buffer.percent is deprecated. Instead,
> > use mapreduce.reduce.input.buffer.percent
> > > 11:14:31,149 INFO
> > org.apache.hadoop.mapreduce.lib.input.FileInputFormat         - Total
> input
> > paths to process : 1
> > > 11:14:31,177 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Scheduling job Flink Java Job at Wed Jan 28 11:14:28 CET 2015.
> > > 11:14:31,188 INFO
> > org.apache.flink.runtime.executiongraph.ExecutionGraph        - Deploying
> > CHAIN DataSource (at createInput(ExecutionEnvironment.java:426)
> > (org.apache.flink.hadoopcompatibility.mapreduce.HadoopInputFormat)) ->
> > FlatMap (FlatMap at main(ThiaziParser.java:37)) (1/1) (attempt #0) to
> > cloud-26
> > > 11:14:31,202 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job c979ea0b2e358fb988f5a2a30ab48786 (Flink Java Job at Wed
> Jan
> > 28 11:14:28 CET 2015) changed to RUNNING.
> > > 11:14:32,525 INFO
> > org.apache.flink.api.common.io.LocatableInputSplitAssigner    - Assigning
> > remote split to host cloud-26
> > > 11:14:32,603 WARN  akka.remote.ReliableDeliverySupervisor
> >         - Association with remote system [akka.tcp://
> > fl...@cloud-26.dima.tu-berlin.de:51449] has failed, address is now gated
> > for [5000] ms. Reason is: [Disassociated].
> > > 11:16:12,574 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job c979ea0b2e358fb988f5a2a30ab48786 (Flink Java Job at Wed
> Jan
> > 28 11:14:28 CET 2015) changed to FAILINGjava.lang.RuntimeException:
> > Requesting the next InputSplit failed.
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >       at
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >       at java.lang.Thread.run(Thread.java:745)
> > > Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after [100 seconds]
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >       at
> > scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >       at
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >       at scala.concurrent.Await$.result(package.scala:107)
> > >       at
> > org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >       ... 4 more
> > > .
> > > 11:16:12,584 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job c979ea0b2e358fb988f5a2a30ab48786 (Flink Java Job at Wed
> Jan
> > 28 11:14:28 CET 2015) changed to FAILEDjava.lang.RuntimeException:
> > Requesting the next InputSplit failed.
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >       at
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >       at java.lang.Thread.run(Thread.java:745)
> > > Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after [100 seconds]
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >       at
> > scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >       at
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >       at scala.concurrent.Await$.result(package.scala:107)
> > >       at
> > org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >       ... 4 more
> > > .
> > > 11:17:53,055 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Received job fe8cebf0874c7f67ac03454de8555fb9 (Flink Java Job at Wed
> Jan
> > 28 11:17:50 CET 2015).
> > > 11:17:53,258 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - io.bytes.per.checksum is deprecated. Instead, use
> > dfs.bytes-per-checksum
> > > 11:17:53,312 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - fs.default.name is deprecated. Instead, use fs.defaultFS
> > > 11:17:53,320 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapreduce.job.counters.limit is deprecated. Instead, use
> > mapreduce.job.counters.max
> > > 11:17:53,347 INFO
> > org.apache.hadoop.mapreduce.lib.input.FileInputFormat         - Total
> input
> > paths to process : 1
> > > 11:17:53,352 INFO
> > org.apache.flink.runtime.executiongraph.ExecutionGraph        - Deploying
> > CHAIN DataSource (at createInput(ExecutionEnvironment.java:426)
> > (org.apache.flink.hadoopcompatibility.mapreduce.HadoopInputFormat)) ->
> > FlatMap (FlatMap at main(ThiaziParser.java:37)) (1/1) (attempt #0) to
> > cloud-31
> > > 11:17:53,353 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Scheduling job Flink Java Job at Wed Jan 28 11:17:50 CET 2015.
> > > 11:17:53,354 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job fe8cebf0874c7f67ac03454de8555fb9 (Flink Java Job at Wed
> Jan
> > 28 11:17:50 CET 2015) changed to RUNNING.
> > > 11:17:54,646 INFO
> > org.apache.flink.api.common.io.LocatableInputSplitAssigner    - Assigning
> > remote split to host cloud-31
> > > 11:17:54,715 WARN  akka.remote.ReliableDeliverySupervisor
> >         - Association with remote system [akka.tcp://
> > fl...@cloud-31.dima.tu-berlin.de:43200] has failed, address is now gated
> > for [5000] ms. Reason is: [Disassociated].
> > > 11:19:34,684 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job fe8cebf0874c7f67ac03454de8555fb9 (Flink Java Job at Wed
> Jan
> > 28 11:17:50 CET 2015) changed to FAILINGjava.lang.RuntimeException:
> > Requesting the next InputSplit failed.
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >       at
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >       at java.lang.Thread.run(Thread.java:745)
> > > Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after [100 seconds]
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >       at
> > scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >       at
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >       at scala.concurrent.Await$.result(package.scala:107)
> > >       at
> > org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >       ... 4 more
> > > .
> > > 11:19:34,697 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job fe8cebf0874c7f67ac03454de8555fb9 (Flink Java Job at Wed
> Jan
> > 28 11:17:50 CET 2015) changed to FAILEDjava.lang.RuntimeException:
> > Requesting the next InputSplit failed.
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >       at
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >       at java.lang.Thread.run(Thread.java:745)
> > > Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after [100 seconds]
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >       at
> > scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >       at
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >       at scala.concurrent.Await$.result(package.scala:107)
> > >       at
> > org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >       ... 4 more
> > > .
> > > 11:42:52,441 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Received job b63a600c2edd93b790f2e984ca987464 (Flink Java Job at Wed
> Jan
> > 28 11:42:50 CET 2015).
> > > 11:42:52,650 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - io.bytes.per.checksum is deprecated. Instead, use
> > dfs.bytes-per-checksum
> > > 11:42:52,705 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - fs.default.name is deprecated. Instead, use fs.defaultFS
> > > 11:42:52,713 INFO  org.apache.hadoop.conf.Configuration.deprecation
> >         - mapreduce.job.counters.limit is deprecated. Instead, use
> > mapreduce.job.counters.max
> > > 11:42:52,740 INFO
> > org.apache.hadoop.mapreduce.lib.input.FileInputFormat         - Total
> input
> > paths to process : 1
> > > 11:42:52,747 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Scheduling job Flink Java Job at Wed Jan 28 11:42:50 CET 2015.
> > > 11:42:52,747 INFO
> > org.apache.flink.runtime.executiongraph.ExecutionGraph        - Deploying
> > CHAIN DataSource (at createInput(ExecutionEnvironment.java:426)
> > (org.apache.flink.hadoopcompatibility.mapreduce.HadoopInputFormat)) ->
> > FlatMap (FlatMap at main(ThiaziParser.java:37)) (1/1) (attempt #0) to
> > cloud-26
> > > 11:42:52,750 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job b63a600c2edd93b790f2e984ca987464 (Flink Java Job at Wed
> Jan
> > 28 11:42:50 CET 2015) changed to RUNNING.
> > > 11:42:52,956 INFO
> > org.apache.flink.api.common.io.LocatableInputSplitAssigner    - Assigning
> > remote split to host cloud-26
> > > 11:42:52,983 WARN  akka.remote.ReliableDeliverySupervisor
> >         - Association with remote system [akka.tcp://
> > fl...@cloud-26.dima.tu-berlin.de:51449] has failed, address is now gated
> > for [5000] ms. Reason is: [Disassociated].
> > > 11:44:32,976 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job b63a600c2edd93b790f2e984ca987464 (Flink Java Job at Wed
> Jan
> > 28 11:42:50 CET 2015) changed to FAILINGjava.lang.RuntimeException:
> > Requesting the next InputSplit failed.
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >       at
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >       at java.lang.Thread.run(Thread.java:745)
> > > Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after [100 seconds]
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >       at
> > scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >       at
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >       at scala.concurrent.Await$.result(package.scala:107)
> > >       at
> > org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >       ... 4 more
> > > .
> > > 11:44:32,984 INFO
> >
> org.apache.flink.yarn.ApplicationMaster$$anonfun$startJobManager$2$$anon$1
> > - Status of job b63a600c2edd93b790f2e984ca987464 (Flink Java Job at Wed
> Jan
> > 28 11:42:50 CET 2015) changed to FAILEDjava.lang.RuntimeException:
> > Requesting the next InputSplit failed.
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >       at
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >       at
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >       at java.lang.Thread.run(Thread.java:745)
> > > Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after [100 seconds]
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >       at
> > scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >       at
> > scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >       at
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >       at scala.concurrent.Await$.result(package.scala:107)
> > >       at
> > org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >       at
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >       ... 4 more
> > > .
> > >
> > > On 28 Jan 2015, at 12:08, Robert Metzger <rmetz...@apache.org> wrote:
> > >
> > >> Hi,
> > >>
> > >> it seems that you are not subscribed to our mailing list, so I had to
> > >> manually accept your mail. Would be good if you could subscribe.
> > >>
> > >> Can you send us also the log output of the JobManager?
> > >> If your YARN cluster has log aggregation activated, you can retrieve
> the
> > >> logs of a stopped YARN session using:
> > >> yarn logs -applicationId <AppId>
> > >>
> > >> watch out for the jobmanager-main.log or so file.
> > >>
> > >> I suspect that there has been an exception on the JobManager.
> > >>
> > >> Best,
> > >> Robert
> > >>
> > >>
> > >>
> > >> On Wed, Jan 28, 2015 at 12:01 PM, Bruecke, Christoph <
> > >> christoph.brue...@campus.tu-berlin.de> wrote:
> > >>
> > >>> Hi,
> > >>>
> > >>> I have written a job that reads a SequenceFile from HDFS using the
> > >>> Hadoop-Compatibility add-on. Doing so results in a TimeoutException.
> > I’m
> > >>> using flink-0.9-SNAPSHOT with PR 342 (
> > >>> https://github.com/apache/flink/pull/342 ). Furthermore I’m running
> > flink
> > >>> on yarn with two TM using flink-yarn-0.9-SNAPSHOT/bin/yarn-session.sh
> > -n 2.
> > >>>
> > >>> Is this a bug or is there something wrong with the configuration?
> > >>>
> > >>> 01/28/2015 11:42:52     Job execution switched to status RUNNING.
> > >>> 01/28/2015 11:42:52     CHAIN DataSource (at
> > >>> createInput(ExecutionEnvironment.java:426)
> > >>> (org.apache.flink.hadoopcompatibility.mapreduce.HadoopInputFormat))
> ->
> > >>> FlatMap (FlatMap at main(ThiaziParser.java:37))(1/1) switched to
> > SCHEDULED
> > >>> 01/28/2015 11:42:52     CHAIN DataSource (at
> > >>> createInput(ExecutionEnvironment.java:426)
> > >>> (org.apache.flink.hadoopcompatibility.mapreduce.HadoopInputFormat))
> ->
> > >>> FlatMap (FlatMap at main(ThiaziParser.java:37))(1/1) switched to
> > DEPLOYING
> > >>> 01/28/2015 11:42:52     CHAIN DataSource (at
> > >>> createInput(ExecutionEnvironment.java:426)
> > >>> (org.apache.flink.hadoopcompatibility.mapreduce.HadoopInputFormat))
> ->
> > >>> FlatMap (FlatMap at main(ThiaziParser.java:37))(1/1) switched to
> > RUNNING
> > >>> 01/28/2015 11:44:32     CHAIN DataSource (at
> > >>> createInput(ExecutionEnvironment.java:426)
> > >>> (org.apache.flink.hadoopcompatibility.mapreduce.HadoopInputFormat))
> ->
> > >>> FlatMap (FlatMap at main(ThiaziParser.java:37))(1/1) switched to
> FAILED
> > >>> java.lang.RuntimeException: Requesting the next InputSplit failed.
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >>>       at java.lang.Thread.run(Thread.java:745)
> > >>> Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after
> > >>> [100 seconds]
> > >>>       at
> > >>> scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >>>       at
> > >>>
> scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >>>       at
> > >>> scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >>>       at
> > >>>
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >>>       at scala.concurrent.Await$.result(package.scala:107)
> > >>>       at
> > >>> org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >>>       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >>>       ... 4 more
> > >>>
> > >>> 01/28/2015 11:44:32     Job execution switched to status FAILING.
> > >>> 01/28/2015 11:44:32     GroupReduce (GroupReduce at
> > >>> main(ThiaziParser.java:40))(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     DataSink(TextOutputFormat (hdfs://
> > >>>
> > cloud-11.dima.tu-berlin.de:60010/user/cbruecke/output/thiazi-seq/authors
> )
> > >>> - UTF-8)(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     CHAIN GroupReduce (GroupReduce at
> > >>> main(ThiaziParser.java:74)) -> Filter (Filter at
> > >>> main(ThiaziParser.java:97))(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     DataSink(TextOutputFormat (hdfs://
> > >>>
> cloud-11.dima.tu-berlin.de:60010/user/cbruecke/output/thiazi-seq/posts)
> > -
> > >>> UTF-8)(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     CHAIN FlatMap (FlatMap at
> > >>> main(ThiaziParser.java:126)) -> Combine(SUM(1), at
> > >>> main(ThiaziParser.java:140)(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     Reduce (SUM(1), at
> > >>> main(ThiaziParser.java:140)(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     DataSink(CsvOutputFormat (path: hdfs://
> > >>>
> >
> cloud-11.dima.tu-berlin.de:60010/user/cbruecke/output/thiazi-seq/wordcount
> > ,
> > >>> delimiter: ,))(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     GroupReduce (GroupReduce at
> > >>> main(ThiaziParser.java:106))(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     DataSink(TextOutputFormat (hdfs://
> > >>>
> > cloud-11.dima.tu-berlin.de:60010/user/cbruecke/output/thiazi-seq/threads
> )
> > >>> - UTF-8)(1/1) switched to CANCELED
> > >>> 01/28/2015 11:44:32     Job execution switched to status FAILED.
> > >>> Error: The program execution failed: java.lang.RuntimeException:
> > >>> Requesting the next InputSplit failed.
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:63)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.operators.DataSourceTask$1.hasNext(DataSourceTask.java:355)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.operators.DataSourceTask.invoke(DataSourceTask.java:154)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.execution.RuntimeEnvironment.run(RuntimeEnvironment.java:204)
> > >>>       at java.lang.Thread.run(Thread.java:745)
> > >>> Caused by: java.util.concurrent.TimeoutException: Futures timed out
> > after
> > >>> [100 seconds]
> > >>>       at
> > >>> scala.concurrent.impl.Promise$DefaultPromise.ready(Promise.scala:219)
> > >>>       at
> > >>>
> scala.concurrent.impl.Promise$DefaultPromise.result(Promise.scala:223)
> > >>>       at
> > >>> scala.concurrent.Await$$anonfun$result$1.apply(package.scala:107)
> > >>>       at
> > >>>
> >
> scala.concurrent.BlockContext$DefaultBlockContext$.blockOn(BlockContext.scala:53)
> > >>>       at scala.concurrent.Await$.result(package.scala:107)
> > >>>       at
> > >>> org.apache.flink.runtime.akka.AkkaUtils$.ask(AkkaUtils.scala:265)
> > >>>       at org.apache.flink.runtime.akka.AkkaUtils.ask(AkkaUtils.scala)
> > >>>       at
> > >>>
> >
> org.apache.flink.runtime.taskmanager.TaskInputSplitProvider.getNextInputSplit(TaskInputSplitProvider.java:56)
> > >>>       ... 4 more
> > >>>
> > >>>
> > >>>
> > >
> >
> >
>

Reply via email to