[jira] [Created] (FLINK-8867) Rocksdb checkpointing failing with fs.default-scheme: hdfs:// config

2018-03-05 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-8867:
---

 Summary: Rocksdb checkpointing failing with fs.default-scheme: 
hdfs:// config
 Key: FLINK-8867
 URL: https://issues.apache.org/jira/browse/FLINK-8867
 Project: Flink
  Issue Type: Bug
  Components: Configuration, State Backends, Checkpointing, YARN
Affects Versions: 1.4.1, 1.4.2
Reporter: Shashank Agarwal
 Fix For: 1.5.0, 1.4.3


In our setup, when we put an entry in our Flink_conf file for default schema.

{code}
fs.default-scheme: hdfs://mydomain.com:8020/flink
{code}

Than application with rocksdb state backend fails with the following exception. 
When we remove this config it works fine. It's working fine with other state 
backends.

{code}
AsynchronousException{java.lang.Exception: Could not materialize checkpoint 1 
for operator order ip stream (1/1).}
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:948)
at 
java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
at java.util.concurrent.FutureTask.run(FutureTask.java:266)
at 
java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1149)
at 
java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:624)
at java.lang.Thread.run(Thread.java:748)
Caused by: java.lang.Exception: Could not materialize checkpoint 1 for operator 
order ip stream (1/1).
... 6 more
Caused by: java.util.concurrent.ExecutionException: 
java.lang.IllegalStateException
at java.util.concurrent.FutureTask.report(FutureTask.java:122)
at java.util.concurrent.FutureTask.get(FutureTask.java:192)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:43)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:894)
... 5 more
Suppressed: java.lang.Exception: Could not properly cancel managed 
keyed state future.
at 
org.apache.flink.streaming.api.operators.OperatorSnapshotResult.cancel(OperatorSnapshotResult.java:91)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.cleanup(StreamTask.java:976)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:939)
... 5 more
Caused by: java.util.concurrent.ExecutionException: 
java.lang.IllegalStateException
at java.util.concurrent.FutureTask.report(FutureTask.java:122)
at java.util.concurrent.FutureTask.get(FutureTask.java:192)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:43)
at 
org.apache.flink.runtime.state.StateUtil.discardStateFuture(StateUtil.java:66)
at 
org.apache.flink.streaming.api.operators.OperatorSnapshotResult.cancel(OperatorSnapshotResult.java:89)
... 7 more
Caused by: java.lang.IllegalStateException
at 
org.apache.flink.util.Preconditions.checkState(Preconditions.java:179)
at 
org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend$RocksDBIncrementalSnapshotOperation.materializeSnapshot(RocksDBKeyedStateBackend.java:926)
at 
org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend$1.call(RocksDBKeyedStateBackend.java:389)
at 
org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend$1.call(RocksDBKeyedStateBackend.java:386)
at java.util.concurrent.FutureTask.run(FutureTask.java:266)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:40)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:894)
... 5 more
[CIRCULAR REFERENCE:java.lang.IllegalStateException]
{code}



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Created] (FLINK-8182) Unable to read hdfs file system directory(which contains sub directories) recursively

2017-12-01 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-8182:
---

 Summary: Unable to read hdfs file system directory(which contains 
sub directories) recursively  
 Key: FLINK-8182
 URL: https://issues.apache.org/jira/browse/FLINK-8182
 Project: Flink
  Issue Type: Improvement
  Components: Streaming
Affects Versions: 1.3.2
Reporter: Shashank Agarwal


Unable to read hdfs file system directory(which contains subdirectories) 
recursively, It works fine when a single directory contains only files but when 
the directory contains subdirectories it dosesn't read subdirectory files.


{code}
streamExecutionEnvironment.readTextFile("HDFS path")
{code}




--
This message was sent by Atlassian JIRA
(v6.4.14#64029)


[jira] [Created] (FLINK-8098) LeaseExpiredException when using FsStateBackend for checkpointing due to multiple mappers tries to access the same file.

2017-11-17 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-8098:
---

 Summary: LeaseExpiredException when using FsStateBackend for 
checkpointing due to multiple mappers tries to access the same file.
 Key: FLINK-8098
 URL: https://issues.apache.org/jira/browse/FLINK-8098
 Project: Flink
  Issue Type: Bug
  Components: State Backends, Checkpointing
Affects Versions: 1.3.2
 Environment: Yarn, HDFS 2.7.3, Kafka, scala streaming API, CEP
Reporter: Shashank Agarwal


I am running streaming application with parallelism 6. I have enabled 
checkpointing(1000). But application gets the crash after 1-2 days. After 
analysing logs i found following trace. 

{code}
2017-11-17 11:19:06,696 WARN  
org.apache.flink.streaming.runtime.tasks.StreamTask   - Could not 
properly clean up the async checkpoint runnable.
java.lang.Exception: Could not properly cancel managed keyed state future.
at 
org.apache.flink.streaming.api.operators.OperatorSnapshotResult.cancel(OperatorSnapshotResult.java:90)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.cleanup(StreamTask.java:1023)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.close(StreamTask.java:983)
at org.apache.flink.util.IOUtils.closeQuietly(IOUtils.java:262)
at org.apache.flink.util.IOUtils.closeAllQuietly(IOUtils.java:251)
at 
org.apache.flink.util.AbstractCloseableRegistry.close(AbstractCloseableRegistry.java:97)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.cancel(StreamTask.java:355)
at 
org.apache.flink.runtime.taskmanager.Task$TaskCanceler.run(Task.java:1463)
at java.lang.Thread.run(Thread.java:745)
Caused by: java.util.concurrent.ExecutionException: java.io.IOException: Could 
not flush and close the file system output stream to 
hdfs://xyz.com:8020/flink/sux/54944cea1f566ee801656e06cdeeabbc/chk-40191/cf145018-0599-4281-b254-96600a4e4965
 in order to obtain the stream state handle
at java.util.concurrent.FutureTask.report(FutureTask.java:122)
at java.util.concurrent.FutureTask.get(FutureTask.java:192)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:43)
at 
org.apache.flink.runtime.state.StateUtil.discardStateFuture(StateUtil.java:85)
at 
org.apache.flink.streaming.api.operators.OperatorSnapshotResult.cancel(OperatorSnapshotResult.java:88)
... 8 more
Caused by: java.io.IOException: Could not flush and close the file system 
output stream to 
hdfs://xyz.com:8020/flink/sux/54944cea1f566ee801656e06cdeeabbc/chk-40191/cf145018-0599-4281-b254-96600a4e4965
 in order to obtain the stream state handle
at 
org.apache.flink.runtime.state.filesystem.FsCheckpointStreamFactory$FsCheckpointStateOutputStream.closeAndGetHandle(FsCheckpointStreamFactory.java:336)
at 
org.apache.flink.runtime.checkpoint.AbstractAsyncSnapshotIOCallable.closeStreamAndGetStateHandle(AbstractAsyncSnapshotIOCallable.java:100)
at 
org.apache.flink.runtime.state.heap.HeapKeyedStateBackend$1.performOperation(HeapKeyedStateBackend.java:351)
at 
org.apache.flink.runtime.state.heap.HeapKeyedStateBackend$1.performOperation(HeapKeyedStateBackend.java:329)
at 
org.apache.flink.runtime.io.async.AbstractAsyncIOCallable.call(AbstractAsyncIOCallable.java:72)
at java.util.concurrent.FutureTask.run(FutureTask.java:266)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:40)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:897)
at 
java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
at java.util.concurrent.FutureTask.run(FutureTask.java:266)
at 
java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1142)
at 
java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:617)
... 1 more
Caused by: 
org.apache.hadoop.ipc.RemoteException(org.apache.hadoop.hdfs.server.namenode.LeaseExpiredException):
 No lease 
flink/sux/54944cea1f566ee801656e06cdeeabbc/chk-40191/cf145018-0599-4281-b254-96600a4e4965
 (inode 812148671): File does not exist. [Lease.  Holder: 
DFSClient_NONMAPREDUCE_1721510813_94, pendingcreates: 161]
at 
org.apache.hadoop.hdfs.server.namenode.FSNamesystem.checkLease(FSNamesystem.java:3659)
at 
org.apache.hadoop.hdfs.server.namenode.FSNamesystem.completeFileInternal(FSNamesystem.java:3749)
at 
org.apache.hadoop.hdfs.server.namenode.FSNamesystem.completeFile(FSNamesystem.java:3716)
at 
org.apache.hadoop.hdfs.server.namenode.NameNodeRpcServer.complete(NameNodeRpcServer.java:911)
at 
org.apache.hadoop.hdfs.protocolPB.ClientNamenodeProtocolServerSideTranslatorPB.complete(ClientNamenodeProtocolServerSideTranslatorPB.java:547)
   

[jira] [Created] (FLINK-7953) Kafka consumer printing error - java.lang.IllegalArgumentException: Invalid offset: -915623761772

2017-10-31 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-7953:
---

 Summary: Kafka consumer printing error - 
java.lang.IllegalArgumentException: Invalid offset: -915623761772
 Key: FLINK-7953
 URL: https://issues.apache.org/jira/browse/FLINK-7953
 Project: Flink
  Issue Type: Bug
  Components: Kafka Connector
Affects Versions: 1.3.2
 Environment: kafka 10.0, yarn
Reporter: Shashank Agarwal
Priority: Minor


As it's printing as Warning and not impacting running program so marked it 
minor. 

{code}
2017-10-31 19:26:09,218 WARN  
org.apache.flink.streaming.connectors.kafka.internal.Kafka09Fetcher  - 
Committing offsets to Kafka failed. This does not compromise Flink's 
checkpoints.
java.lang.IllegalArgumentException: Invalid offset: -915623761772
at 
org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.sendOffsetCommitRequest(ConsumerCoordinator.java:687)
at 
org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.doCommitOffsetsAsync(ConsumerCoordinator.java:531)
at 
org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.commitOffsetsAsync(ConsumerCoordinator.java:499)
at 
org.apache.kafka.clients.consumer.KafkaConsumer.commitAsync(KafkaConsumer.java:1181)
at 
org.apache.flink.streaming.connectors.kafka.internal.KafkaConsumerThread.run(KafkaConsumerThread.java:223)
2017-10-31 19:26:09,223 ERROR 
org.apache.flink.streaming.connectors.kafka.FlinkKafkaConsumerBase  - Async 
Kafka commit failed.
java.lang.IllegalArgumentException: Invalid offset: -915623761772
at 
org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.sendOffsetCommitRequest(ConsumerCoordinator.java:687)
at 
org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.doCommitOffsetsAsync(ConsumerCoordinator.java:531)
at 
org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.commitOffsetsAsync(ConsumerCoordinator.java:499)
at 
org.apache.kafka.clients.consumer.KafkaConsumer.commitAsync(KafkaConsumer.java:1181)
at 
org.apache.flink.streaming.connectors.kafka.internal.KafkaConsumerThread.run(KafkaConsumerThread.java:223)
{code}




--
This message was sent by Atlassian JIRA
(v6.4.14#64029)


[jira] [Created] (FLINK-7760) Restore failing from external checkpointing metadata.

2017-10-04 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-7760:
---

 Summary: Restore failing from external checkpointing metadata.
 Key: FLINK-7760
 URL: https://issues.apache.org/jira/browse/FLINK-7760
 Project: Flink
  Issue Type: Bug
  Components: CEP, State Backends, Checkpointing
Affects Versions: 1.3.2
 Environment: Yarn, Flink 1.3.2, HDFS,  FsStateBackend
Reporter: Shashank Agarwal


My job failed due to failure of cassandra. I have enabled 
ExternalizedCheckpoints. But when job tried to restore from that checkpoint 
it's failing continuously with following error.

{code:java}
2017-10-04 09:39:20,611 INFO  org.apache.flink.runtime.taskmanager.Task 
- KeyedCEPPatternOperator -> Map (1/2) 
(8ff7913f820ead571c8b54ccc6b16045) switched from RUNNING to FAILED.
java.lang.IllegalStateException: Could not initialize keyed state backend.
at 
org.apache.flink.streaming.api.operators.AbstractStreamOperator.initKeyedState(AbstractStreamOperator.java:321)
at 
org.apache.flink.streaming.api.operators.AbstractStreamOperator.initializeState(AbstractStreamOperator.java:217)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.initializeOperators(StreamTask.java:676)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.initializeState(StreamTask.java:663)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:252)
at org.apache.flink.runtime.taskmanager.Task.run(Task.java:702)
at java.lang.Thread.run(Thread.java:745)
Caused by: java.io.StreamCorruptedException: invalid type code: 00
at 
java.io.ObjectInputStream$BlockDataInputStream.readBlockHeader(ObjectInputStream.java:2519)
at 
java.io.ObjectInputStream$BlockDataInputStream.refill(ObjectInputStream.java:2553)
at 
java.io.ObjectInputStream$BlockDataInputStream.skipBlockData(ObjectInputStream.java:2455)
at java.io.ObjectInputStream.skipCustomData(ObjectInputStream.java:1951)
at 
java.io.ObjectInputStream.readNonProxyDesc(ObjectInputStream.java:1621)
at java.io.ObjectInputStream.readClassDesc(ObjectInputStream.java:1518)
at 
java.io.ObjectInputStream.readNonProxyDesc(ObjectInputStream.java:1623)
at java.io.ObjectInputStream.readClassDesc(ObjectInputStream.java:1518)
at 
java.io.ObjectInputStream.readOrdinaryObject(ObjectInputStream.java:1774)
at java.io.ObjectInputStream.readObject0(ObjectInputStream.java:1351)
at java.io.ObjectInputStream.readObject(ObjectInputStream.java:371)
at 
org.apache.flink.cep.nfa.NFA$NFASerializer.deserializeCondition(NFA.java:1211)
at 
org.apache.flink.cep.nfa.NFA$NFASerializer.deserializeStates(NFA.java:1169)
at org.apache.flink.cep.nfa.NFA$NFASerializer.deserialize(NFA.java:957)
at org.apache.flink.cep.nfa.NFA$NFASerializer.deserialize(NFA.java:852)
at 
org.apache.flink.runtime.state.heap.StateTableByKeyGroupReaders$StateTableByKeyGroupReaderV2V3.readMappingsInKeyGroup(StateTableByKeyGroupReaders.java:132)
at 
org.apache.flink.runtime.state.heap.HeapKeyedStateBackend.restorePartitionedState(HeapKeyedStateBackend.java:518)
at 
org.apache.flink.runtime.state.heap.HeapKeyedStateBackend.restore(HeapKeyedStateBackend.java:397)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.createKeyedStateBackend(StreamTask.java:772)
at 
org.apache.flink.streaming.api.operators.AbstractStreamOperator.initKeyedState(AbstractStreamOperator.java:311)
... 6 more
{code}

I have tried to start new job also after failure with parameter {code:java} -s 
[checkpoint meta data path]{code}





--
This message was sent by Atlassian JIRA
(v6.4.14#64029)


[jira] [Created] (FLINK-7756) RocksDB state backend Checkpointing (Async and Incremental) is not working with CEP.

2017-10-03 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-7756:
---

 Summary: RocksDB state backend Checkpointing (Async and 
Incremental)  is not working with CEP.
 Key: FLINK-7756
 URL: https://issues.apache.org/jira/browse/FLINK-7756
 Project: Flink
  Issue Type: Bug
  Components: CEP, State Backends, Checkpointing, Streaming
Affects Versions: 1.3.2
 Environment: Flink 1.3.2, Yarn, HDFS, RocksDB backend
Reporter: Shashank Agarwal


When i try to use RocksDBStateBackend on my staging cluster (which is using 
HDFS as file system) it crashes. But When i use FsStateBackend on staging 
(which is using HDFS as file system) it is working fine.

On local with local file system it's working fine in both cases.

Please check attached logs. I have around 20-25 tasks in my app.


{code:java}
2017-09-29 14:21:31,639 INFO  
org.apache.flink.streaming.connectors.fs.bucketing.BucketingSink  - No state to 
restore for the BucketingSink (taskIdx=0).
2017-09-29 14:21:31,640 INFO  
org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend  - 
Initializing RocksDB keyed state backend from snapshot.
2017-09-29 14:21:32,020 INFO  
org.apache.flink.streaming.connectors.fs.bucketing.BucketingSink  - No state to 
restore for the BucketingSink (taskIdx=1).
2017-09-29 14:21:32,022 INFO  
org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend  - 
Initializing RocksDB keyed state backend from snapshot.
2017-09-29 14:21:32,078 INFO  com.datastax.driver.core.NettyUtil
- Found Netty's native epoll transport in the classpath, using it
2017-09-29 14:21:34,177 INFO  org.apache.flink.runtime.taskmanager.Task 
- Attempting to fail task externally Co-Flat Map (1/2) 
(b879f192c4e8aae6671cdafb3a24c00a).
2017-09-29 14:21:34,177 INFO  org.apache.flink.runtime.taskmanager.Task 
- Attempting to fail task externally Map (2/2) 
(1ea5aef6ccc7031edc6b37da2912d90b).
2017-09-29 14:21:34,178 INFO  org.apache.flink.runtime.taskmanager.Task 
- Attempting to fail task externally Co-Flat Map (2/2) 
(4bac8e764c67520d418a4c755be23d4d).
2017-09-29 14:21:34,178 INFO  org.apache.flink.runtime.taskmanager.Task 
- Co-Flat Map (1/2) (b879f192c4e8aae6671cdafb3a24c00a) switched 
from RUNNING to FAILED.
AsynchronousException{java.lang.Exception: Could not materialize checkpoint 2 
for operator Co-Flat Map (1/2).}
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:970)
at 
java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
at java.util.concurrent.FutureTask.run(FutureTask.java:266)
at 
java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1142)
at 
java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:617)
at java.lang.Thread.run(Thread.java:745)
Caused by: java.lang.Exception: Could not materialize checkpoint 2 for operator 
Co-Flat Map (1/2).
... 6 more
Caused by: java.util.concurrent.ExecutionException: 
java.lang.IllegalStateException
at java.util.concurrent.FutureTask.report(FutureTask.java:122)
at java.util.concurrent.FutureTask.get(FutureTask.java:192)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:43)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:897)
... 5 more
Suppressed: java.lang.Exception: Could not properly cancel managed 
keyed state future.
at 
org.apache.flink.streaming.api.operators.OperatorSnapshotResult.cancel(OperatorSnapshotResult.java:90)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.cleanup(StreamTask.java:1023)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:961)
... 5 more
Caused by: java.util.concurrent.ExecutionException: 
java.lang.IllegalStateException
at java.util.concurrent.FutureTask.report(FutureTask.java:122)
at java.util.concurrent.FutureTask.get(FutureTask.java:192)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:43)
at 
org.apache.flink.runtime.state.StateUtil.discardStateFuture(StateUtil.java:85)
at 
org.apache.flink.streaming.api.operators.OperatorSnapshotResult.cancel(OperatorSnapshotResult.java:88)
... 7 more
Caused by: java.lang.IllegalStateException
at 
org.apache.flink.util.Preconditions.checkState(Preconditions.java:179)
at 
org.apache.flink.contrib.streaming.state.RocksDBKeyedStateBackend$RocksDBIncrementalSnapshotOperation.materializeSnapshot(RocksDBKeyedStateBackend.java:878)

[jira] [Created] (FLINK-7484) com.esotericsoftware.kryo.KryoException: java.lang.IndexOutOfBoundsException: Index: 7, Size: 5

2017-08-21 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-7484:
---

 Summary: com.esotericsoftware.kryo.KryoException: 
java.lang.IndexOutOfBoundsException: Index: 7, Size: 5
 Key: FLINK-7484
 URL: https://issues.apache.org/jira/browse/FLINK-7484
 Project: Flink
  Issue Type: Bug
  Components: DataStream API, Scala API
Affects Versions: 1.3.2
 Environment: Flink 1.3.2 , Yarn Cluster, FsStateBackend
Reporter: Shashank Agarwal


I am using many CEP's and Global Window. I am getting following error sometimes 
and application  crashes. I have checked logically there's no flow in the 
program. Here ItemPojo is a Pojo class and we are using 
java.utill.List[ItemPojo]. We are using Scala DataStream API please find 
attached logs.


{code}
2017-08-17 10:04:12,814 INFO  org.apache.flink.runtime.taskmanager.Task 
- TriggerWindow(GlobalWindows(), 
ListStateDescriptor{serializer=org.apache.flink.api.common.typeutils.base.ListSerializer@6d36aa3c},
 co.thirdwatch.trigger.TransactionTrigger@5707c1cb, 
WindowedStream.apply(WindowedStream.scala:582)) -> Flat Map -> Map -> Sink: 
Saving CSV Features Sink (1/2) (06c0d4d231bc620ba9e7924b9b0da8d1) switched from 
RUNNING to FAILED.
com.esotericsoftware.kryo.KryoException: java.lang.IndexOutOfBoundsException: 
Index: 7, Size: 5
Serialization trace:
category (co.thirdwatch.pojo.ItemPojo)
underlying (scala.collection.convert.Wrappers$SeqWrapper)
at 
com.esotericsoftware.kryo.serializers.ObjectField.read(ObjectField.java:125)
at 
com.esotericsoftware.kryo.serializers.FieldSerializer.read(FieldSerializer.java:528)
at com.esotericsoftware.kryo.Kryo.readClassAndObject(Kryo.java:761)
at com.twitter.chill.TraversableSerializer.read(Traversable.scala:43)
at com.twitter.chill.TraversableSerializer.read(Traversable.scala:21)
at com.esotericsoftware.kryo.Kryo.readObject(Kryo.java:679)
at 
com.esotericsoftware.kryo.serializers.ObjectField.read(ObjectField.java:106)
at 
com.esotericsoftware.kryo.serializers.FieldSerializer.read(FieldSerializer.java:528)
at com.esotericsoftware.kryo.Kryo.readObject(Kryo.java:657)
at 
org.apache.flink.api.java.typeutils.runtime.kryo.KryoSerializer.copy(KryoSerializer.java:190)
at 
org.apache.flink.api.scala.typeutils.CaseClassSerializer.copy(CaseClassSerializer.scala:101)
at 
org.apache.flink.api.scala.typeutils.CaseClassSerializer.copy(CaseClassSerializer.scala:32)
at 
org.apache.flink.runtime.state.ArrayListSerializer.copy(ArrayListSerializer.java:74)
at 
org.apache.flink.runtime.state.ArrayListSerializer.copy(ArrayListSerializer.java:34)
at 
org.apache.flink.runtime.state.heap.CopyOnWriteStateTable.get(CopyOnWriteStateTable.java:279)
at 
org.apache.flink.runtime.state.heap.CopyOnWriteStateTable.get(CopyOnWriteStateTable.java:296)
at 
org.apache.flink.runtime.state.heap.HeapListState.add(HeapListState.java:77)
at 
org.apache.flink.streaming.runtime.operators.windowing.WindowOperator.processElement(WindowOperator.java:442)
at 
org.apache.flink.streaming.runtime.io.StreamInputProcessor.processInput(StreamInputProcessor.java:206)
at 
org.apache.flink.streaming.runtime.tasks.OneInputStreamTask.run(OneInputStreamTask.java:69)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:263)
at org.apache.flink.runtime.taskmanager.Task.run(Task.java:702)
at java.lang.Thread.run(Thread.java:745)
Caused by: java.lang.IndexOutOfBoundsException: Index: 7, Size: 5
at java.util.ArrayList.rangeCheck(ArrayList.java:653)
at java.util.ArrayList.get(ArrayList.java:429)
at 
com.esotericsoftware.kryo.util.MapReferenceResolver.getReadObject(MapReferenceResolver.java:42)
at com.esotericsoftware.kryo.Kryo.readReferenceOrNull(Kryo.java:805)
at com.esotericsoftware.kryo.Kryo.readObjectOrNull(Kryo.java:728)
at 
com.esotericsoftware.kryo.serializers.ObjectField.read(ObjectField.java:113)
... 22 more
2017-08-17 10:04:12,816 INFO  org.apache.flink.runtime.taskmanager.Task 
- Freeing task resources for TriggerWindow(GlobalWindows(), 
ListStateDescriptor{serializer=org.apache.flink.api.common.typeutils.base.ListSerializer@6d36aa3c},
 co.thirdwatch.trigger.TransactionTrigger@5707c1cb, 
WindowedStream.apply(WindowedStream.scala:582)) -> Flat Map -> Map -> Sink: 
Saving CSV Features Sink (1/2) (06c0d4d231bc620ba9e7924b9b0da8d1).
2017-08-17 10:04:12,816 INFO  org.apache.flink.runtime.taskmanager.Task 
- Ensuring all FileSystem streams are closed for task 
TriggerWindow(GlobalWindows(), 
ListStateDescriptor{serializer=org.apache.flink.api.common.typeutils.base.ListSerializer@6d36aa3c},
 co.thirdwatch.trigger.TransactionTrigger@5707c1cb, 

[jira] [Created] (FLINK-6993) Not reading recursive files in Batch by using readTextFile when file name contains _ in starting.

2017-06-23 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-6993:
---

 Summary: Not reading recursive files in Batch by using 
readTextFile when file name contains _ in starting.
 Key: FLINK-6993
 URL: https://issues.apache.org/jira/browse/FLINK-6993
 Project: Flink
  Issue Type: Bug
  Components: Batch Connectors and Input/Output Formats
Affects Versions: 1.3.0
Reporter: Shashank Agarwal
Priority: Critical
 Fix For: 1.3.2


When i try to read files from a folder using using readTextFile in batch and 
using recursive.file.enumeration, It's not reading the files when file name 
contains _ in starting. But when i removed the _ from start it's working fine. 

It also working fine in case of direct path of single file not working with 
Directory path. For replicate the issue :

{code}
import org.apache.flink.api.scala.{DataSet, ExecutionEnvironment}
import org.apache.flink.configuration.Configuration

object CSVMerge {

  def main(args: Array[String]): Unit = {

val env = ExecutionEnvironment.getExecutionEnvironment

// create a configuration object
val parameters = new Configuration

// set the recursive enumeration parameter
parameters.setBoolean("recursive.file.enumeration", true)

val stream = env.readTextFile("file:///Users/data")
  .withParameters(parameters)

stream.print()

  }
}
{code}

When you put 2-3 Text files with name like 1.txt, 2.txt etc. in data folder 
it's working fine. But when we put _1.txt, _2.txt file it's not working.

Flink BucketingSink in stream by default put _ before the file names. So unable 
to read Sinked files from DataStream.



--
This message was sent by Atlassian JIRA
(v6.4.14#64029)


[jira] [Created] (FLINK-6954) Flink 1.3 checkpointing failing with KeyedCEPPatternOperator

2017-06-20 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-6954:
---

 Summary: Flink 1.3 checkpointing failing with 
KeyedCEPPatternOperator
 Key: FLINK-6954
 URL: https://issues.apache.org/jira/browse/FLINK-6954
 Project: Flink
  Issue Type: Bug
  Components: CEP, DataStream API, State Backends, Checkpointing
Affects Versions: 1.3.0
 Environment: yarn, flink 1.3, HDFS
Reporter: Shashank Agarwal


After upgrading to Flink 1.3 Checkpointing is not working, it's failing again 
and again. Check operator state. I have checked with both Rocks DB state 
backend and FS state backend. Check stack trace. 
{code}
java.lang.Exception: Could not perform checkpoint 1 for operator 
KeyedCEPPatternOperator -> Map (6/6).
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.triggerCheckpointOnBarrier(StreamTask.java:550)
at 
org.apache.flink.streaming.runtime.io.BarrierBuffer.notifyCheckpoint(BarrierBuffer.java:378)
at 
org.apache.flink.streaming.runtime.io.BarrierBuffer.processBarrier(BarrierBuffer.java:281)
at 
org.apache.flink.streaming.runtime.io.BarrierBuffer.getNextNonBlocked(BarrierBuffer.java:183)
at 
org.apache.flink.streaming.runtime.io.StreamInputProcessor.processInput(StreamInputProcessor.java:213)
at 
org.apache.flink.streaming.runtime.tasks.OneInputStreamTask.run(OneInputStreamTask.java:69)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:262)
at org.apache.flink.runtime.taskmanager.Task.run(Task.java:702)
at java.lang.Thread.run(Thread.java:745)
Caused by: java.lang.Exception: Could not complete snapshot 1 for operator 
KeyedCEPPatternOperator -> Map (6/6).
at 
org.apache.flink.streaming.api.operators.AbstractStreamOperator.snapshotState(AbstractStreamOperator.java:406)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$CheckpointingOperation.checkpointStreamOperator(StreamTask.java:1157)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$CheckpointingOperation.executeCheckpointing(StreamTask.java:1089)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.checkpointState(StreamTask.java:653)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.performCheckpoint(StreamTask.java:589)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask.triggerCheckpointOnBarrier(StreamTask.java:542)
... 8 more
Caused by: java.lang.UnsupportedOperationException
at 
org.apache.flink.api.scala.typeutils.TraversableSerializer.snapshotConfiguration(TraversableSerializer.scala:155)
at 
org.apache.flink.api.common.typeutils.CompositeTypeSerializerConfigSnapshot.(CompositeTypeSerializerConfigSnapshot.java:53)
at 
org.apache.flink.api.scala.typeutils.OptionSerializer$OptionSerializerConfigSnapshot.(OptionSerializer.scala:139)
at 
org.apache.flink.api.scala.typeutils.OptionSerializer.snapshotConfiguration(OptionSerializer.scala:104)
at 
org.apache.flink.api.scala.typeutils.OptionSerializer.snapshotConfiguration(OptionSerializer.scala:28)
at 
org.apache.flink.api.common.typeutils.CompositeTypeSerializerConfigSnapshot.(CompositeTypeSerializerConfigSnapshot.java:53)
at 
org.apache.flink.api.java.typeutils.runtime.TupleSerializerConfigSnapshot.(TupleSerializerConfigSnapshot.java:45)
at 
org.apache.flink.api.java.typeutils.runtime.TupleSerializerBase.snapshotConfiguration(TupleSerializerBase.java:132)
at 
org.apache.flink.api.java.typeutils.runtime.TupleSerializerBase.snapshotConfiguration(TupleSerializerBase.java:39)
at 
org.apache.flink.api.common.typeutils.CompositeTypeSerializerConfigSnapshot.(CompositeTypeSerializerConfigSnapshot.java:53)
at 
org.apache.flink.api.common.typeutils.base.CollectionSerializerConfigSnapshot.(CollectionSerializerConfigSnapshot.java:39)
at 
org.apache.flink.api.common.typeutils.base.ListSerializer.snapshotConfiguration(ListSerializer.java:183)
at 
org.apache.flink.api.common.typeutils.base.ListSerializer.snapshotConfiguration(ListSerializer.java:47)
at 
org.apache.flink.api.common.typeutils.CompositeTypeSerializerConfigSnapshot.(CompositeTypeSerializerConfigSnapshot.java:53)
at 
org.apache.flink.api.common.typeutils.base.MapSerializerConfigSnapshot.(MapSerializerConfigSnapshot.java:38)
at 
org.apache.flink.runtime.state.HashMapSerializer.snapshotConfiguration(HashMapSerializer.java:210)
at 
org.apache.flink.runtime.state.RegisteredKeyedBackendStateMetaInfo.snapshot(RegisteredKeyedBackendStateMetaInfo.java:71)
at 
org.apache.flink.runtime.state.heap.HeapKeyedStateBackend.snapshot(HeapKeyedStateBackend.java:267)
at 
org.apache.flink.streaming.api.operators.AbstractStreamOperator.snapshotState(AbstractStreamOperator.java:396)
... 13 more

{code}



--
This message was sent by 

[jira] [Created] (FLINK-6460) Build Fat Jar failed while building with SBT Assembly flink 1.2.1

2017-05-05 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-6460:
---

 Summary: Build Fat Jar failed while building with SBT Assembly 
flink 1.2.1
 Key: FLINK-6460
 URL: https://issues.apache.org/jira/browse/FLINK-6460
 Project: Flink
  Issue Type: Bug
  Components: Build System, Scala API
Affects Versions: 1.2.1
 Environment: Ubuntu 16.04, Scala, SBT, Flink 1.2.1
Reporter: Shashank Agarwal


While Creating SBT assembly from command 
{code}sbt clean assembly{code}

Getting error deduplicate: different file contents. error log :

{code}
[error] (root/*:assembly) deduplicate: different file contents found in the 
following:
[error] 
/Users/shashank/.ivy2/cache/io.netty/netty-handler/jars/netty-handler-4.0.33.Final.jar:META-INF/io.netty.versions.properties
[error] 
/Users/shashank/.ivy2/cache/io.netty/netty-buffer/jars/netty-buffer-4.0.33.Final.jar:META-INF/io.netty.versions.properties
[error] 
/Users/shashank/.ivy2/cache/io.netty/netty-common/jars/netty-common-4.0.33.Final.jar:META-INF/io.netty.versions.properties
[error] 
/Users/shashank/.ivy2/cache/io.netty/netty-transport/jars/netty-transport-4.0.33.Final.jar:META-INF/io.netty.versions.properties
[error] 
/Users/shashank/.ivy2/cache/io.netty/netty-codec/jars/netty-codec-4.0.33.Final.jar:META-INF/io.netty.versions.properties
[error] Total time: 66 s, completed 5 May, 2017 2:47:03 PM
{code}



--
This message was sent by Atlassian JIRA
(v6.3.15#6346)


[jira] [Created] (FLINK-6321) RocksDB state backend Checkpointing is not working with KeydCEP.

2017-04-18 Thread Shashank Agarwal (JIRA)
Shashank Agarwal created FLINK-6321:
---

 Summary: RocksDB state backend Checkpointing is not working with 
KeydCEP.
 Key: FLINK-6321
 URL: https://issues.apache.org/jira/browse/FLINK-6321
 Project: Flink
  Issue Type: Bug
  Components: CEP, State Backends, Checkpointing
Affects Versions: 1.2.0
 Environment: yarn-cluster, RocksDB State backend, Checkpointing every 
1000 ms
Reporter: Shashank Agarwal


Checkpointing is not working with RocksDBStateBackend when using CEP. It's 
working fine with FsStateBackend and MemoryStateBackend. Application failing 
every-time.

'''
04/18/2017 21:53:20 Job execution switched to status FAILING.
AsynchronousException{java.lang.Exception: Could not materialize checkpoint 46 
for operator KeyedCEPPatternOperator -> Map (1/4).}
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:980)
at 
java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511)
at java.util.concurrent.FutureTask.run(FutureTask.java:266)
at 
java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1142)
at 
java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:617)
at java.lang.Thread.run(Thread.java:745)
Caused by: java.lang.Exception: Could not materialize checkpoint 46 for 
operator KeyedCEPPatternOperator -> Map (1/4).
... 6 more
Caused by: java.util.concurrent.CancellationException
at java.util.concurrent.FutureTask.report(FutureTask.java:121)
at java.util.concurrent.FutureTask.get(FutureTask.java:192)
at 
org.apache.flink.util.FutureUtil.runIfNotDoneAndGet(FutureUtil.java:40)
at 
org.apache.flink.streaming.runtime.tasks.StreamTask$AsyncCheckpointRunnable.run(StreamTask.java:915)
... 5 more

'''





--
This message was sent by Atlassian JIRA
(v6.3.15#6346)