Uploaded image for project: 'Flink'
  1. Flink
  2. FLINK-12301

Scala value classes inside case classes cannot be serialized anymore in Flink 1.8.0

    XMLWordPrintableJSON

Details

    Description

      There is a regression in Flink 1.8.0 compared to 1.7.2: Scala value classes cannot be serialized anymore as a case class attribute.

      Some short example code:

      package com.example.valueclassissue
      
      import org.apache.flink.streaming.api.scala._
      
      object ValueClassExample extends App {
        val env: StreamExecutionEnvironment = StreamExecutionEnvironment.getExecutionEnvironment
      
        val measurements = env.fromCollection(Seq(Measurement(1, new DegreeCelsius(32.5f))))
        measurements.print()
      
        env.execute()
      }
      
      class DegreeCelsius(val value: Float) extends AnyVal {
        override def toString: String = s"$value °C"
      }
      
      case class Measurement(i: Int, temperature: DegreeCelsius)
      

      While with Flink 1.7.2 the program outputs 3> Measurement(1,32.5 °C) as expected, in Flink 1.8.0 an exception is thrown:

      java.io.IOException: Failed to deserialize an element from the source. If you are using user-defined serialization (Value and Writable types), check the serialization functions.
      Serializer is org.apache.flink.api.scala.typeutils.ScalaCaseClassSerializer@466b6f83
      	at org.apache.flink.streaming.api.functions.source.FromElementsFunction.run(FromElementsFunction.java:158)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:93)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:57)
      	at org.apache.flink.streaming.runtime.tasks.SourceStreamTask.run(SourceStreamTask.java:97)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
      	at java.lang.Thread.run(Thread.java:748)
      

      Full log:

      2019-04-23T09:33:48.485Z [main] INFO org.apache.flink.api.java.typeutils.TypeExtractor - class com.example.valueclassissue.DegreeCelsius does not contain a setter for field value
      2019-04-23T09:33:48.487Z [main] INFO org.apache.flink.api.java.typeutils.TypeExtractor - Class class com.example.valueclassissue.DegreeCelsius cannot be used as a POJO type because not all fields are valid POJO fields, and must be processed as GenericType. Please read the Flink documentation on "Data Types & Serialization" for details of the effect on performance.
      2019-04-23T09:33:49.594Z [main] INFO org.apache.flink.streaming.api.environment.LocalStreamEnvironment - Running job on local embedded Flink mini cluster
      2019-04-23T09:33:49.616Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Starting Flink Mini Cluster
      2019-04-23T09:33:49.618Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Starting Metrics Registry
      2019-04-23T09:33:49.665Z [main] INFO org.apache.flink.runtime.metrics.MetricRegistryImpl - No metrics reporter configured, no metrics will be exposed/reported.
      2019-04-23T09:33:49.665Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Starting RPC Service(s)
      2019-04-23T09:33:49.896Z [flink-akka.actor.default-dispatcher-3] INFO akka.event.slf4j.Slf4jLogger - Slf4jLogger started
      2019-04-23T09:33:49.913Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Trying to start actor system at :0
      2019-04-23T09:33:49.952Z [flink-metrics-2] INFO akka.event.slf4j.Slf4jLogger - Slf4jLogger started
      2019-04-23T09:33:50.001Z [flink-metrics-2] INFO akka.remote.Remoting - Starting remoting
      2019-04-23T09:33:50.139Z [flink-metrics-2] INFO akka.remote.Remoting - Remoting started; listening on addresses :[akka.tcp://flink-metrics@127.0.1.1:36651]
      2019-04-23T09:33:50.144Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Actor system started at akka.tcp://flink-metrics@127.0.1.1:36651
      2019-04-23T09:33:50.146Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Starting high-availability services
      2019-04-23T09:33:50.155Z [main] INFO org.apache.flink.runtime.blob.BlobServer - Created BLOB server storage directory /tmp/blobStore-0e1c3305-1e6e-47a0-885a-8c82bb5ae09f
      2019-04-23T09:33:50.158Z [main] INFO org.apache.flink.runtime.blob.BlobServer - Started BLOB server at 0.0.0.0:40817 - max concurrent requests: 50 - max backlog: 1000
      2019-04-23T09:33:50.161Z [main] INFO org.apache.flink.runtime.blob.PermanentBlobCache - Created BLOB cache storage directory /tmp/blobStore-e3046023-c398-4a42-88f4-20949f7876ce
      2019-04-23T09:33:50.162Z [main] INFO org.apache.flink.runtime.blob.TransientBlobCache - Created BLOB cache storage directory /tmp/blobStore-ef6f2625-afcc-4937-90ba-a981af59bce4
      2019-04-23T09:33:50.162Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Starting 1 TaskManger(s)
      2019-04-23T09:33:50.164Z [main] INFO org.apache.flink.runtime.taskexecutor.TaskManagerRunner - Starting TaskManager with ResourceID: ce7b0b20-a06b-4135-aa6b-7a051cfc3672
      2019-04-23T09:33:50.217Z [main] INFO org.apache.flink.runtime.taskexecutor.TaskManagerServices - Temporary file directory '/tmp': total 97 GB, usable 60 GB (61.86% usable)
      2019-04-23T09:33:50.325Z [main] INFO org.apache.flink.runtime.io.network.buffer.NetworkBufferPool - Allocated 246 MB for network buffer pool (number of memory segments: 7882, bytes per segment: 32768).
      2019-04-23T09:33:50.328Z [main] INFO org.apache.flink.runtime.io.network.NetworkEnvironment - Starting the network environment and its components.
      2019-04-23T09:33:50.329Z [main] INFO org.apache.flink.runtime.taskexecutor.TaskManagerServices - Limiting managed memory to 0.7 of the currently free heap space (1543 MB), memory will be allocated lazily.
      2019-04-23T09:33:50.331Z [main] INFO org.apache.flink.runtime.io.disk.iomanager.IOManager - I/O manager uses directory /tmp/flink-io-f952881a-df88-4b6f-84fe-1e83412f120b for spill files.
      2019-04-23T09:33:50.377Z [main] INFO org.apache.flink.runtime.taskexecutor.TaskManagerConfiguration - Messages have a max timeout of 10000 ms
      2019-04-23T09:33:50.387Z [main] INFO org.apache.flink.runtime.rpc.akka.AkkaRpcService - Starting RPC endpoint for org.apache.flink.runtime.taskexecutor.TaskExecutor at akka://flink/user/taskmanager_0 .
      2019-04-23T09:33:50.400Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Start job leader service.
      2019-04-23T09:33:50.401Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.filecache.FileCache - User file cache uses directory /tmp/flink-dist-cache-da57d8de-8359-4f8b-838d-88094398b7cd
      2019-04-23T09:33:50.430Z [main] INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint - Starting rest endpoint.
      2019-04-23T09:33:50.573Z [main] WARN org.apache.flink.runtime.webmonitor.WebMonitorUtils - Log file environment variable 'log.file' is not set.
      2019-04-23T09:33:50.573Z [main] WARN org.apache.flink.runtime.webmonitor.WebMonitorUtils - JobManager log files are unavailable in the web dashboard. Log file location not found in environment variable 'log.file' or configuration key 'Key: 'web.log.path' , default: null (fallback keys: [{key=jobmanager.web.log.path, isDeprecated=true}])'.
      2019-04-23T09:33:50.580Z [main] INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint - Failed to load web based job submission extension. Probable reason: flink-runtime-web is not in the classpath.
      2019-04-23T09:33:50.715Z [main] INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint - Rest endpoint listening at localhost:39621
      2019-04-23T09:33:50.716Z [main] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Proposing leadership to contender org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint@25f723b0 @ http://localhost:39621
      2019-04-23T09:33:50.718Z [mini-cluster-io-thread-1] INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint - http://localhost:39621 was granted leadership with leaderSessionID=848a2169-0638-4284-b5b1-c74561a5016d
      2019-04-23T09:33:50.718Z [mini-cluster-io-thread-1] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Received confirmation of leadership for leader http://localhost:39621 , session=848a2169-0638-4284-b5b1-c74561a5016d
      2019-04-23T09:33:50.728Z [main] INFO org.apache.flink.runtime.rpc.akka.AkkaRpcService - Starting RPC endpoint for org.apache.flink.runtime.resourcemanager.StandaloneResourceManager at akka://flink/user/resourcemanager .
      2019-04-23T09:33:50.737Z [main] INFO org.apache.flink.runtime.rpc.akka.AkkaRpcService - Starting RPC endpoint for org.apache.flink.runtime.dispatcher.StandaloneDispatcher at akka://flink/user/dispatcher .
      2019-04-23T09:33:50.745Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Proposing leadership to contender org.apache.flink.runtime.resourcemanager.StandaloneResourceManager@586de9c6 @ akka://flink/user/resourcemanager
      2019-04-23T09:33:50.745Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Proposing leadership to contender org.apache.flink.runtime.dispatcher.StandaloneDispatcher@171bdca8 @ akka://flink/user/dispatcher
      2019-04-23T09:33:50.748Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - ResourceManager akka://flink/user/resourcemanager was granted leadership with fencing token 8d0000744239b25d92a24a84384e421b
      2019-04-23T09:33:50.748Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.slotmanager.SlotManager - Starting the SlotManager.
      2019-04-23T09:33:50.750Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Received confirmation of leadership for leader akka://flink/user/resourcemanager , session=92a24a84-384e-421b-8d00-00744239b25d
      2019-04-23T09:33:50.751Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Dispatcher akka://flink/user/dispatcher was granted leadership with fencing token 3be1cf58-f256-464c-a503-0130f1d6653a
      2019-04-23T09:33:50.755Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Flink Mini Cluster started successfully
      2019-04-23T09:33:50.759Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Connecting to ResourceManager akka://flink/user/resourcemanager(8d0000744239b25d92a24a84384e421b).
      2019-04-23T09:33:50.760Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Recovering all persisted jobs.
      2019-04-23T09:33:50.761Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Received confirmation of leadership for leader akka://flink/user/dispatcher , session=3be1cf58-f256-464c-a503-0130f1d6653a
      2019-04-23T09:33:50.767Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Resolved ResourceManager address, beginning registration
      2019-04-23T09:33:50.768Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Registration at ResourceManager attempt 1 (timeout=100ms)
      2019-04-23T09:33:50.771Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Registering TaskManager with ResourceID ce7b0b20-a06b-4135-aa6b-7a051cfc3672 (akka://flink/user/taskmanager_0) at ResourceManager
      2019-04-23T09:33:50.772Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Successful registration at resource manager akka://flink/user/resourcemanager under registration id 57ee60b35a4b387afe62844e5dd075e5.
      2019-04-23T09:33:50.775Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Received JobGraph submission 0934b4e32657180f004df5cf4e003ab4 (Flink Streaming Job).
      2019-04-23T09:33:50.775Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Submitting job 0934b4e32657180f004df5cf4e003ab4 (Flink Streaming Job).
      2019-04-23T09:33:50.788Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.rpc.akka.AkkaRpcService - Starting RPC endpoint for org.apache.flink.runtime.jobmaster.JobMaster at akka://flink/user/jobmanager_1 .
      2019-04-23T09:33:50.796Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.jobmaster.JobMaster - Initializing job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4).
      2019-04-23T09:33:50.800Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.jobmaster.JobMaster - Using restart strategy NoRestartStrategy for Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4).
      2019-04-23T09:33:50.827Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Job recovers via failover strategy: full graph restart
      2019-04-23T09:33:50.847Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.jobmaster.JobMaster - Running initialization on master for job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4).
      2019-04-23T09:33:50.847Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.jobmaster.JobMaster - Successfully ran initialization on master in 0 ms.
      2019-04-23T09:33:50.866Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.jobmaster.JobMaster - No state backend has been configured, using default (Memory / JobManager) MemoryStateBackend (data in heap memory / checkpoints to JobManager) (checkpoints: 'null', savepoints: 'null', asynchronous: TRUE, maxStateSize: 5242880)
      2019-04-23T09:33:50.876Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Proposing leadership to contender org.apache.flink.runtime.jobmaster.JobManagerRunner@6667cd6 @ akka://flink/user/jobmanager_1
      2019-04-23T09:33:50.877Z [mini-cluster-io-thread-4] INFO org.apache.flink.runtime.jobmaster.JobManagerRunner - JobManager runner for job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4) was granted leadership with session id ed72f40e-a416-4efc-856d-27e694574e2f at akka://flink/user/jobmanager_1.
      2019-04-23T09:33:50.879Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.jobmaster.JobMaster - Starting execution of job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4) under job master id 856d27e694574e2fed72f40ea4164efc.
      2019-04-23T09:33:50.879Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4) switched from state CREATED to RUNNING.
      2019-04-23T09:33:50.882Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) switched from CREATED to SCHEDULED.
      2019-04-23T09:33:50.890Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Cannot serve slot request, no ResourceManager connected. Adding as pending request [SlotRequestId{6a6e14d6bca90867458577365cb9721e}]
      2019-04-23T09:33:50.894Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from CREATED to SCHEDULED.
      2019-04-23T09:33:50.895Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from CREATED to SCHEDULED.
      2019-04-23T09:33:50.895Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from CREATED to SCHEDULED.
      2019-04-23T09:33:50.895Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from CREATED to SCHEDULED.
      2019-04-23T09:33:50.898Z [jobmanager-future-thread-1] INFO org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService - Received confirmation of leadership for leader akka://flink/user/jobmanager_1 , session=ed72f40e-a416-4efc-856d-27e694574e2f
      2019-04-23T09:33:50.898Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.jobmaster.JobMaster - Connecting to ResourceManager akka://flink/user/resourcemanager(8d0000744239b25d92a24a84384e421b)
      2019-04-23T09:33:50.900Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.jobmaster.JobMaster - Resolved ResourceManager address, beginning registration
      2019-04-23T09:33:50.900Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.jobmaster.JobMaster - Registration at ResourceManager attempt 1 (timeout=100ms)
      2019-04-23T09:33:50.902Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Registering job manager 856d27e694574e2fed72f40ea4164efc@akka://flink/user/jobmanager_1 for job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.908Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Registered job manager 856d27e694574e2fed72f40ea4164efc@akka://flink/user/jobmanager_1 for job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.909Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.jobmaster.JobMaster - JobManager successfully registered at ResourceManager, leader id: 8d0000744239b25d92a24a84384e421b.
      2019-04-23T09:33:50.910Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Requesting new slot [SlotRequestId{6a6e14d6bca90867458577365cb9721e}] and profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} from resource manager.
      2019-04-23T09:33:50.911Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Request slot with profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} for job 0934b4e32657180f004df5cf4e003ab4 with allocation id 16865b3b201cc25ae128095d117c2e7d.
      2019-04-23T09:33:50.911Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Receive slot request 16865b3b201cc25ae128095d117c2e7d for job 0934b4e32657180f004df5cf4e003ab4 from resource manager with leader id 8d0000744239b25d92a24a84384e421b.
      2019-04-23T09:33:50.912Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Allocated slot for 16865b3b201cc25ae128095d117c2e7d.
      2019-04-23T09:33:50.912Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Add job 0934b4e32657180f004df5cf4e003ab4 for job leader monitoring.
      2019-04-23T09:33:50.914Z [mini-cluster-io-thread-3] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Try to register at job manager akka://flink/user/jobmanager_1 with leader id ed72f40e-a416-4efc-856d-27e694574e2f.
      2019-04-23T09:33:50.915Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Resolved JobManager address, beginning registration
      2019-04-23T09:33:50.915Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Registration at JobManager attempt 1 (timeout=100ms)
      2019-04-23T09:33:50.918Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Successful registration at job manager akka://flink/user/jobmanager_1 for job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.918Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Establish JobManager connection for job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.920Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Offer reserved slots to the leader of job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.923Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Requesting new slot [SlotRequestId{0889151458281fff281d1f40b8949362}] and profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} from resource manager.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Requesting new slot [SlotRequestId{6048f2d28bc988068b4f3c09c96e5bf4}] and profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} from resource manager.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Request slot with profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} for job 0934b4e32657180f004df5cf4e003ab4 with allocation id 04c413ac1d7681b769798f365e4a8565.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Requesting new slot [SlotRequestId{af9271e58daaf7760d2bad798c764943}] and profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} from resource manager.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Request slot with profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} for job 0934b4e32657180f004df5cf4e003ab4 with allocation id 4c11ede5a7f087d8653710365e1af854.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Receive slot request 04c413ac1d7681b769798f365e4a8565 for job 0934b4e32657180f004df5cf4e003ab4 from resource manager with leader id 8d0000744239b25d92a24a84384e421b.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Request slot with profile ResourceProfile{cpuCores=-1.0, heapMemoryInMB=-1, directMemoryInMB=0, nativeMemoryInMB=0, networkMemoryInMB=0} for job 0934b4e32657180f004df5cf4e003ab4 with allocation id e31bbfd59504d04ff607286275900171.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Allocated slot for 04c413ac1d7681b769798f365e4a8565.
      2019-04-23T09:33:50.924Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Offer reserved slots to the leader of job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.925Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Received repeated offer for slot [16865b3b201cc25ae128095d117c2e7d]. Ignoring.
      2019-04-23T09:33:50.925Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Receive slot request 4c11ede5a7f087d8653710365e1af854 for job 0934b4e32657180f004df5cf4e003ab4 from resource manager with leader id 8d0000744239b25d92a24a84384e421b.
      2019-04-23T09:33:50.925Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Allocated slot for 4c11ede5a7f087d8653710365e1af854.
      2019-04-23T09:33:50.925Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Offer reserved slots to the leader of job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.925Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Received repeated offer for slot [04c413ac1d7681b769798f365e4a8565]. Ignoring.
      2019-04-23T09:33:50.925Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 16865b3b201cc25ae128095d117c2e7d.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Received repeated offer for slot [16865b3b201cc25ae128095d117c2e7d]. Ignoring.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Receive slot request e31bbfd59504d04ff607286275900171 for job 0934b4e32657180f004df5cf4e003ab4 from resource manager with leader id 8d0000744239b25d92a24a84384e421b.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Allocated slot for e31bbfd59504d04ff607286275900171.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Offer reserved slots to the leader of job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 04c413ac1d7681b769798f365e4a8565.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Received repeated offer for slot [04c413ac1d7681b769798f365e4a8565]. Ignoring.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 16865b3b201cc25ae128095d117c2e7d.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 04c413ac1d7681b769798f365e4a8565.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) switched from SCHEDULED to DEPLOYING.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 16865b3b201cc25ae128095d117c2e7d.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 4c11ede5a7f087d8653710365e1af854.
      2019-04-23T09:33:50.926Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Deploying Source: Collection Source (1/1) (attempt #0) to ce7b0b20-a06b-4135-aa6b-7a051cfc3672 @ localhost (dataPort=-1)
      2019-04-23T09:33:50.930Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from SCHEDULED to DEPLOYING.
      2019-04-23T09:33:50.930Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Deploying Sink: Print to Std. Out (1/4) (attempt #0) to ce7b0b20-a06b-4135-aa6b-7a051cfc3672 @ localhost (dataPort=-1)
      2019-04-23T09:33:50.932Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from SCHEDULED to DEPLOYING.
      2019-04-23T09:33:50.932Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Deploying Sink: Print to Std. Out (2/4) (attempt #0) to ce7b0b20-a06b-4135-aa6b-7a051cfc3672 @ localhost (dataPort=-1)
      2019-04-23T09:33:50.932Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from SCHEDULED to DEPLOYING.
      2019-04-23T09:33:50.932Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Deploying Sink: Print to Std. Out (3/4) (attempt #0) to ce7b0b20-a06b-4135-aa6b-7a051cfc3672 @ localhost (dataPort=-1)
      2019-04-23T09:33:50.932Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from SCHEDULED to DEPLOYING.
      2019-04-23T09:33:50.932Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Deploying Sink: Print to Std. Out (4/4) (attempt #0) to ce7b0b20-a06b-4135-aa6b-7a051cfc3672 @ localhost (dataPort=-1)
      2019-04-23T09:33:50.932Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Received repeated offer for slot [4c11ede5a7f087d8653710365e1af854]. Ignoring.
      2019-04-23T09:33:50.943Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Received task Source: Collection Source (1/1).
      2019-04-23T09:33:50.943Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 04c413ac1d7681b769798f365e4a8565.
      2019-04-23T09:33:50.943Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) switched from CREATED to DEPLOYING.
      2019-04-23T09:33:50.944Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot e31bbfd59504d04ff607286275900171.
      2019-04-23T09:33:50.944Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.slot.TaskSlotTable - Activate slot 4c11ede5a7f087d8653710365e1af854.
      2019-04-23T09:33:50.944Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Creating FileSystem stream leak safety net for task Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) [DEPLOYING]
      2019-04-23T09:33:50.948Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Loading JAR files for task Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) [DEPLOYING].
      2019-04-23T09:33:50.949Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Registering task at network: Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) [DEPLOYING].
      2019-04-23T09:33:50.949Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Received task Sink: Print to Std. Out (1/4).
      2019-04-23T09:33:50.951Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from CREATED to DEPLOYING.
      2019-04-23T09:33:50.951Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Creating FileSystem stream leak safety net for task Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) [DEPLOYING]
      2019-04-23T09:33:50.951Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Received task Sink: Print to Std. Out (2/4).
      2019-04-23T09:33:50.951Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Loading JAR files for task Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) [DEPLOYING].
      2019-04-23T09:33:50.952Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Registering task at network: Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) [DEPLOYING].
      2019-04-23T09:33:50.953Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Received task Sink: Print to Std. Out (3/4).
      2019-04-23T09:33:50.953Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from CREATED to DEPLOYING.
      2019-04-23T09:33:50.953Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Creating FileSystem stream leak safety net for task Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) [DEPLOYING]
      2019-04-23T09:33:50.953Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Loading JAR files for task Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) [DEPLOYING].
      2019-04-23T09:33:50.954Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Registering task at network: Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) [DEPLOYING].
      2019-04-23T09:33:50.955Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Received task Sink: Print to Std. Out (4/4).
      2019-04-23T09:33:50.955Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from CREATED to DEPLOYING.
      2019-04-23T09:33:50.955Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Creating FileSystem stream leak safety net for task Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) [DEPLOYING]
      2019-04-23T09:33:50.955Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Loading JAR files for task Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) [DEPLOYING].
      2019-04-23T09:33:50.956Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Registering task at network: Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) [DEPLOYING].
      2019-04-23T09:33:50.963Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from CREATED to DEPLOYING.
      2019-04-23T09:33:50.963Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.964Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.963Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.964Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Creating FileSystem stream leak safety net for task Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) [DEPLOYING]
      2019-04-23T09:33:50.965Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.967Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Loading JAR files for task Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) [DEPLOYING].
      2019-04-23T09:33:50.965Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.streaming.runtime.tasks.StreamTask - No state backend has been configured, using default (Memory / JobManager) MemoryStateBackend (data in heap memory / checkpoints to JobManager) (checkpoints: 'null', savepoints: 'null', asynchronous: TRUE, maxStateSize: 5242880)
      2019-04-23T09:33:50.965Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.967Z [Source: Collection Source (1/1)] INFO org.apache.flink.streaming.runtime.tasks.StreamTask - No state backend has been configured, using default (Memory / JobManager) MemoryStateBackend (data in heap memory / checkpoints to JobManager) (checkpoints: 'null', savepoints: 'null', asynchronous: TRUE, maxStateSize: 5242880)
      2019-04-23T09:33:50.967Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.965Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.streaming.runtime.tasks.StreamTask - No state backend has been configured, using default (Memory / JobManager) MemoryStateBackend (data in heap memory / checkpoints to JobManager) (checkpoints: 'null', savepoints: 'null', asynchronous: TRUE, maxStateSize: 5242880)
      2019-04-23T09:33:50.966Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.967Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.streaming.runtime.tasks.StreamTask - No state backend has been configured, using default (Memory / JobManager) MemoryStateBackend (data in heap memory / checkpoints to JobManager) (checkpoints: 'null', savepoints: 'null', asynchronous: TRUE, maxStateSize: 5242880)
      2019-04-23T09:33:50.967Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.968Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Registering task at network: Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) [DEPLOYING].
      2019-04-23T09:33:50.969Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.969Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from DEPLOYING to RUNNING.
      2019-04-23T09:33:50.969Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.streaming.runtime.tasks.StreamTask - No state backend has been configured, using default (Memory / JobManager) MemoryStateBackend (data in heap memory / checkpoints to JobManager) (checkpoints: 'null', savepoints: 'null', asynchronous: TRUE, maxStateSize: 5242880)
      2019-04-23T09:33:51.028Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) switched from RUNNING to FAILED.
      java.io.IOException: Failed to deserialize an element from the source. If you are using user-defined serialization (Value and Writable types), check the serialization functions.
      Serializer is org.apache.flink.api.scala.typeutils.ScalaCaseClassSerializer@466b6f83
      	at org.apache.flink.streaming.api.functions.source.FromElementsFunction.run(FromElementsFunction.java:158)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:93)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:57)
      	at org.apache.flink.streaming.runtime.tasks.SourceStreamTask.run(SourceStreamTask.java:97)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
      	at java.lang.Thread.run(Thread.java:748)
      2019-04-23T09:33:51.029Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Freeing task resources for Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2).
      2019-04-23T09:33:51.045Z [Source: Collection Source (1/1)] INFO org.apache.flink.runtime.taskmanager.Task - Ensuring all FileSystem streams are closed for task Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) [FAILED]
      2019-04-23T09:33:51.059Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Un-registering task and sending final execution state FAILED to JobManager for task Source: Collection Source 833716fb73726ad792a24c184d5312b2.
      2019-04-23T09:33:51.064Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Source: Collection Source (1/1) (833716fb73726ad792a24c184d5312b2) switched from RUNNING to FAILED.
      java.io.IOException: Failed to deserialize an element from the source. If you are using user-defined serialization (Value and Writable types), check the serialization functions.
      Serializer is org.apache.flink.api.scala.typeutils.ScalaCaseClassSerializer@466b6f83
      	at org.apache.flink.streaming.api.functions.source.FromElementsFunction.run(FromElementsFunction.java:158)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:93)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:57)
      	at org.apache.flink.streaming.runtime.tasks.SourceStreamTask.run(SourceStreamTask.java:97)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
      	at java.lang.Thread.run(Thread.java:748)
      2019-04-23T09:33:51.064Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4) switched from state RUNNING to FAILING.
      java.io.IOException: Failed to deserialize an element from the source. If you are using user-defined serialization (Value and Writable types), check the serialization functions.
      Serializer is org.apache.flink.api.scala.typeutils.ScalaCaseClassSerializer@466b6f83
      	at org.apache.flink.streaming.api.functions.source.FromElementsFunction.run(FromElementsFunction.java:158)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:93)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:57)
      	at org.apache.flink.streaming.runtime.tasks.SourceStreamTask.run(SourceStreamTask.java:97)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
      	at java.lang.Thread.run(Thread.java:748)
      2019-04-23T09:33:51.065Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.066Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Discarding the results produced by task execution 833716fb73726ad792a24c184d5312b2.
      2019-04-23T09:33:51.066Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Attempting to cancel task Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9).
      2019-04-23T09:33:51.066Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.066Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Triggering cancellation of task code Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9).
      2019-04-23T09:33:51.068Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.069Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.072Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.073Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Freeing task resources for Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9).
      2019-04-23T09:33:51.073Z [Sink: Print to Std. Out (1/4)] INFO org.apache.flink.runtime.taskmanager.Task - Ensuring all FileSystem streams are closed for task Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) [CANCELED]
      2019-04-23T09:33:51.075Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.081Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Attempting to cancel task Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125).
      2019-04-23T09:33:51.082Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.082Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Triggering cancellation of task code Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125).
      2019-04-23T09:33:51.084Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.084Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Freeing task resources for Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125).
      2019-04-23T09:33:51.085Z [Sink: Print to Std. Out (2/4)] INFO org.apache.flink.runtime.taskmanager.Task - Ensuring all FileSystem streams are closed for task Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) [CANCELED]
      2019-04-23T09:33:51.084Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Un-registering task and sending final execution state CANCELED to JobManager for task Sink: Print to Std. Out 6982b1cd1147229d264344ec41df12a9.
      2019-04-23T09:33:51.087Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Attempting to cancel task Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93).
      2019-04-23T09:33:51.088Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.088Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Triggering cancellation of task code Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93).
      2019-04-23T09:33:51.089Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (1/4) (6982b1cd1147229d264344ec41df12a9) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.099Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Attempting to cancel task Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9).
      2019-04-23T09:33:51.099Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from RUNNING to CANCELING.
      2019-04-23T09:33:51.100Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskmanager.Task - Triggering cancellation of task code Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9).
      2019-04-23T09:33:51.100Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.100Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Freeing task resources for Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93).
      2019-04-23T09:33:51.100Z [Sink: Print to Std. Out (3/4)] INFO org.apache.flink.runtime.taskmanager.Task - Ensuring all FileSystem streams are closed for task Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) [CANCELED]
      2019-04-23T09:33:51.103Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.103Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Freeing task resources for Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9).
      2019-04-23T09:33:51.104Z [Sink: Print to Std. Out (4/4)] INFO org.apache.flink.runtime.taskmanager.Task - Ensuring all FileSystem streams are closed for task Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) [CANCELED]
      2019-04-23T09:33:51.106Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Un-registering task and sending final execution state CANCELED to JobManager for task Sink: Print to Std. Out fcccdf96f9411000517ecc55b620f125.
      2019-04-23T09:33:51.107Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Un-registering task and sending final execution state CANCELED to JobManager for task Sink: Print to Std. Out 09fdc88b7fe9efe5eeb0d30327eb8c93.
      2019-04-23T09:33:51.108Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (2/4) (fcccdf96f9411000517ecc55b620f125) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.109Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (3/4) (09fdc88b7fe9efe5eeb0d30327eb8c93) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.112Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Un-registering task and sending final execution state CANCELED to JobManager for task Sink: Print to Std. Out 27df5777eb4ebd374f003634a00a7cd9.
      2019-04-23T09:33:51.113Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Sink: Print to Std. Out (4/4) (27df5777eb4ebd374f003634a00a7cd9) switched from CANCELING to CANCELED.
      2019-04-23T09:33:51.114Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Try to restart or fail the job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4) if no longer possible.
      2019-04-23T09:33:51.114Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4) switched from state FAILING to FAILED.
      java.io.IOException: Failed to deserialize an element from the source. If you are using user-defined serialization (Value and Writable types), check the serialization functions.
      Serializer is org.apache.flink.api.scala.typeutils.ScalaCaseClassSerializer@466b6f83
      	at org.apache.flink.streaming.api.functions.source.FromElementsFunction.run(FromElementsFunction.java:158)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:93)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:57)
      	at org.apache.flink.streaming.runtime.tasks.SourceStreamTask.run(SourceStreamTask.java:97)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
      	at java.lang.Thread.run(Thread.java:748)
      2019-04-23T09:33:51.114Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.executiongraph.ExecutionGraph - Could not restart the job Flink Streaming Job (0934b4e32657180f004df5cf4e003ab4) because the restart strategy prevented it.
      java.io.IOException: Failed to deserialize an element from the source. If you are using user-defined serialization (Value and Writable types), check the serialization functions.
      Serializer is org.apache.flink.api.scala.typeutils.ScalaCaseClassSerializer@466b6f83
      	at org.apache.flink.streaming.api.functions.source.FromElementsFunction.run(FromElementsFunction.java:158)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:93)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:57)
      	at org.apache.flink.streaming.runtime.tasks.SourceStreamTask.run(SourceStreamTask.java:97)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
      	at java.lang.Thread.run(Thread.java:748)
      2019-04-23T09:33:51.114Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.checkpoint.CheckpointCoordinator - Stopping checkpoint coordinator for job 0934b4e32657180f004df5cf4e003ab4.
      2019-04-23T09:33:51.114Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.checkpoint.StandaloneCompletedCheckpointStore - Shutting down
      2019-04-23T09:33:51.124Z [main] INFO org.apache.flink.runtime.minicluster.MiniCluster - Shutting down Flink Mini Cluster
      2019-04-23T09:33:51.124Z [main] INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint - Shutting down rest endpoint.
      2019-04-23T09:33:51.124Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Job 0934b4e32657180f004df5cf4e003ab4 reached globally terminal state FAILED.
      2019-04-23T09:33:51.124Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Stopping TaskExecutor akka://flink/user/taskmanager_0.
      2019-04-23T09:33:51.125Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Stop job leader service.
      2019-04-23T09:33:51.128Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.state.TaskExecutorLocalStateStoresManager - Shutting down TaskExecutorLocalStateStoresManager.
      2019-04-23T09:33:51.136Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.JobMaster - Stopping the JobMaster for job Flink Streaming Job(0934b4e32657180f004df5cf4e003ab4).
      2019-04-23T09:33:51.137Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Suspending SlotPool.
      2019-04-23T09:33:51.137Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.JobMaster - Close ResourceManager connection a7d3b0f5395dbabb2f7c22f3c62765c1: JobManager is shutting down..
      2019-04-23T09:33:51.137Z [flink-akka.actor.default-dispatcher-5] INFO org.apache.flink.runtime.jobmaster.slotpool.SlotPoolImpl - Stopping SlotPool.
      2019-04-23T09:33:51.137Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Disconnect job manager 856d27e694574e2fed72f40ea4164efc@akka://flink/user/jobmanager_1 for job 0934b4e32657180f004df5cf4e003ab4 from the resource manager.
      2019-04-23T09:33:51.149Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.io.disk.iomanager.IOManager - I/O manager removed spill file directory /tmp/flink-io-f952881a-df88-4b6f-84fe-1e83412f120b
      2019-04-23T09:33:51.150Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.io.network.NetworkEnvironment - Shutting down the network environment and its components.
      2019-04-23T09:33:51.154Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.taskexecutor.JobLeaderService - Stop job leader service.
      2019-04-23T09:33:51.154Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.filecache.FileCache - removed file cache directory /tmp/flink-dist-cache-da57d8de-8359-4f8b-838d-88094398b7cd
      2019-04-23T09:33:51.154Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.taskexecutor.TaskExecutor - Stopped TaskExecutor akka://flink/user/taskmanager_0.
      2019-04-23T09:33:51.156Z [ForkJoinPool.commonPool-worker-1] INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint - Removing cache directory /tmp/flink-web-ui
      2019-04-23T09:33:51.156Z [ForkJoinPool.commonPool-worker-1] INFO org.apache.flink.runtime.dispatcher.DispatcherRestEndpoint - Shut down complete.
      2019-04-23T09:33:51.159Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.StandaloneResourceManager - Shut down cluster because application is in CANCELED, diagnostics DispatcherResourceManagerComponent has been closed..
      2019-04-23T09:33:51.160Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Stopping dispatcher akka://flink/user/dispatcher.
      2019-04-23T09:33:51.160Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.slotmanager.SlotManager - Closing the SlotManager.
      2019-04-23T09:33:51.160Z [flink-akka.actor.default-dispatcher-4] INFO org.apache.flink.runtime.resourcemanager.slotmanager.SlotManager - Suspending the SlotManager.
      2019-04-23T09:33:51.160Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Stopping all currently running jobs of dispatcher akka://flink/user/dispatcher.
      2019-04-23T09:33:51.161Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.rest.handler.legacy.backpressure.StackTraceSampleCoordinator - Shutting down stack trace sample coordinator.
      2019-04-23T09:33:51.161Z [flink-akka.actor.default-dispatcher-2] INFO org.apache.flink.runtime.dispatcher.StandaloneDispatcher - Stopped dispatcher akka://flink/user/dispatcher.
      2019-04-23T09:33:51.171Z [flink-metrics-2] INFO akka.remote.RemoteActorRefProvider$RemotingTerminator - Shutting down remote daemon.
      2019-04-23T09:33:51.172Z [flink-metrics-2] INFO akka.remote.RemoteActorRefProvider$RemotingTerminator - Remote daemon shut down; proceeding with flushing remote transports.
      2019-04-23T09:33:51.191Z [flink-metrics-2] INFO akka.remote.RemoteActorRefProvider$RemotingTerminator - Remoting shut down.
      2019-04-23T09:33:51.206Z [flink-metrics-2] INFO org.apache.flink.runtime.rpc.akka.AkkaRpcService - Stopping Akka RPC service.
      2019-04-23T09:33:51.223Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.blob.PermanentBlobCache - Shutting down BLOB cache
      2019-04-23T09:33:51.223Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.blob.TransientBlobCache - Shutting down BLOB cache
      2019-04-23T09:33:51.223Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.blob.BlobServer - Stopped BLOB server at 0.0.0.0:40817
      2019-04-23T09:33:51.223Z [flink-akka.actor.default-dispatcher-3] INFO org.apache.flink.runtime.rpc.akka.AkkaRpcService - Stopped Akka RPC service.
      Exception in thread "main" org.apache.flink.runtime.client.JobExecutionException: Job execution failed.
      	at org.apache.flink.runtime.jobmaster.JobResult.toJobExecutionResult(JobResult.java:146)
      	at org.apache.flink.runtime.minicluster.MiniCluster.executeJobBlocking(MiniCluster.java:638)
      	at org.apache.flink.streaming.api.environment.LocalStreamEnvironment.execute(LocalStreamEnvironment.java:123)
      	at org.apache.flink.streaming.api.environment.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.java:1509)
      	at org.apache.flink.streaming.api.scala.StreamExecutionEnvironment.execute(StreamExecutionEnvironment.scala:645)
      	at com.example.valueclassissue.ValueClassExample$.delayedEndpoint$com$gotomeeting$xqm$streamprocessor$ValueClassExample$1(ValueClassExample.scala:11)
      	at com.example.valueclassissue.ValueClassExample$delayedInit$body.apply(ValueClassExample.scala:5)
      	at scala.Function0$class.apply$mcV$sp(Function0.scala:34)
      	at scala.runtime.AbstractFunction0.apply$mcV$sp(AbstractFunction0.scala:12)
      	at scala.App$$anonfun$main$1.apply(App.scala:76)
      	at scala.App$$anonfun$main$1.apply(App.scala:76)
      	at scala.collection.immutable.List.foreach(List.scala:392)
      	at scala.collection.generic.TraversableForwarder$class.foreach(TraversableForwarder.scala:35)
      	at scala.App$class.main(App.scala:76)
      	at com.example.valueclassissue.ValueClassExample$.main(ValueClassExample.scala:5)
      	at com.example.valueclassissue.ValueClassExample.main(ValueClassExample.scala)
      Caused by: java.io.IOException: Failed to deserialize an element from the source. If you are using user-defined serialization (Value and Writable types), check the serialization functions.
      Serializer is org.apache.flink.api.scala.typeutils.ScalaCaseClassSerializer@466b6f83
      	at org.apache.flink.streaming.api.functions.source.FromElementsFunction.run(FromElementsFunction.java:158)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:93)
      	at org.apache.flink.streaming.api.operators.StreamSource.run(StreamSource.java:57)
      	at org.apache.flink.streaming.runtime.tasks.SourceStreamTask.run(SourceStreamTask.java:97)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:300)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:711)
      	at java.lang.Thread.run(Thread.java:748)
      

      Attachments

        Issue Links

          Activity

            People

              aljoscha Aljoscha Krettek
              hiddenbit Michael
              Votes:
              0 Vote for this issue
              Watchers:
              7 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved:

                Time Tracking

                  Estimated:
                  Original Estimate - Not Specified
                  Not Specified
                  Remaining:
                  Remaining Estimate - 0h
                  0h
                  Logged:
                  Time Spent - 20m
                  20m