Uploaded image for project: 'Flume'
  1. Flume
  2. FLUME-3107

When batchSize of sink greater than transactionCapacity of File Channel, Flume can produce endless data

    XMLWordPrintableJSON

Details

    • Bug
    • Status: Resolved
    • Major
    • Resolution: Resolved
    • 1.7.0
    • 1.9.0
    • File Channel
    • None

    Description

      This problem is the similar as it in FLUME-3106.Flume can produce endless data When batchSize of sink greater than transactionCapacity of File Channel, you can try it with the following config:

      agent.sources = src1
      agent.sinks = sink1
      agent.channels = ch2
      
      agent.sources.src1.type = spooldir
      agent.sources.src1.channels = ch2
      agent.sources.src1.spoolDir = /home/kafka/flumeSpooldir
      agent.sources.src1.fileHeader = false
      agent.sources.src1.batchSize = 5
      
      agent.channels.ch2.type=file
      agent.channels.ch2.capacity=100
      agent.channels.ch2.checkpointDir=/home/kafka/flumefilechannel/checkpointDir
      agent.channels.ch2.dataDirs=/home/kafka/flumefilechannel/dataDirs
      agent.channels.ch2.transactionCapacity=5
      
      agent.sinks.sink1.type = hdfs
      agent.sinks.sink1.channel = ch2
      agent.sinks.sink1.hdfs.path = hdfs://kafka1:9000/flume/
      agent.sinks.sink1.hdfs.rollInterval=1
      agent.sinks.sink1.hdfs.fileType = DataStream
      agent.sinks.sink1.hdfs.writeFormat = Text
      agent.sinks.sink1.hdfs.batchSize = 10
      

      Exceptions like this:

      17/06/09 17:16:18 ERROR flume.SinkRunner: Unable to deliver event. Exception follows.
      org.apache.flume.EventDeliveryException: org.apache.flume.ChannelException: Take list for FileBackedTransaction, capacity 5 full, consider
      committing more frequently, increasing capacity, or increasing thread count. [channel=ch2]
              at org.apache.flume.sink.hdfs.HDFSEventSink.process(HDFSEventSink.java:451)
              at org.apache.flume.sink.DefaultSinkProcessor.process(DefaultSinkProcessor.java:67)
              at org.apache.flume.SinkRunner$PollingRunner.run(SinkRunner.java:145)
              at java.lang.Thread.run(Thread.java:745)
      Caused by: org.apache.flume.ChannelException: Take list for FileBackedTransaction, capacity 5 full, consider committing more frequently, in
      creasing capacity, or increasing thread count. [channel=ch2]
              at org.apache.flume.channel.file.FileChannel$FileBackedTransaction.doTake(FileChannel.java:531)
              at org.apache.flume.channel.BasicTransactionSemantics.take(BasicTransactionSemantics.java:113)
              at org.apache.flume.channel.BasicChannelSemantics.take(BasicChannelSemantics.java:95)
              at org.apache.flume.sink.hdfs.HDFSEventSink.process(HDFSEventSink.java:362)
              ... 3 more
      

      Attachments

        1. FLUME-3107-0.patch
          1 kB
          Yongxi Zhang

        Issue Links

          Activity

            People

              Unassigned Unassigned
              xyz2277 Yongxi Zhang
              Votes:
              0 Vote for this issue
              Watchers:
              2 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved: