You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@impala.apache.org by "Vuk Ercegovac (JIRA)" <ji...@apache.org> on 2018/04/18 21:34:00 UTC

[jira] [Created] (IMPALA-6880) test_bloom_wait_time fails

Vuk Ercegovac created IMPALA-6880:
-------------------------------------

             Summary: test_bloom_wait_time fails
                 Key: IMPALA-6880
                 URL: https://issues.apache.org/jira/browse/IMPALA-6880
             Project: IMPALA
          Issue Type: Bug
          Components: Backend
    Affects Versions: Impala 3.0
            Reporter: Vuk Ercegovac


{noformat}
query_test/test_runtime_filters.py:90: in test_bloom_wait_time
    self.run_test_case('QueryTest/bloom_filters_wait', vector)
common/impala_test_suite.py:444: in run_test_case
    verify_runtime_profile(test_section['RUNTIME_PROFILE'], result.runtime_profile)
common/test_result_verifier.py:560: in verify_runtime_profile
    actual))
E   AssertionError: Did not find matches for lines in runtime profile:
E   EXPECTED LINES:
E   row_regex: .*0 of 1 Runtime Filter Published, 1 Disabled.*
E   
E   ACTUAL PROFILE:
E   Query (id=31479a87f057d480:fb4b93dc00000000):
E     Summary:
E       Session ID: e54d8da4b1fc778d:f545112eedcbaca3
E       Session Type: BEESWAX
E       Start Time: 2018-04-18 04:35:25.168216000
E       End Time: 
E       Query Type: QUERY
E       Query State: FINISHED
E       Query Status: OK
E       Impala Version: impalad version 3.0.0-SNAPSHOT RELEASE (build eaf66172df113dbf10cdb0a08a2bc51e4077ca38)
E       User: jenkins
E       Connected User: jenkins
E       Delegated User: 
E       Network Address: 127.0.0.1:42339
E       Default Db: functional_text_snap
E       Sql Statement: with l as (select * from tpch.lineitem UNION ALL select * from tpch.lineitem)
E   select STRAIGHT_JOIN count(*) from (select * from tpch.lineitem a LIMIT 1) a
E       join (select * from l LIMIT 500000) b on a.l_orderkey = -b.l_orderkey
E       Coordinator: ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22000
E       Query Options (set by configuration): ABORT_ON_ERROR=1,EXEC_SINGLE_NODE_ROWS_THRESHOLD=0,RUNTIME_FILTER_WAIT_TIME_MS=600000,RUNTIME_FILTER_MAX_SIZE=65536,DISABLE_CODEGEN_ROWS_THRESHOLD=0
E       Query Options (set by configuration and planner): ABORT_ON_ERROR=1,EXEC_SINGLE_NODE_ROWS_THRESHOLD=0,RUNTIME_FILTER_WAIT_TIME_MS=600000,MT_DOP=0,RUNTIME_FILTER_MAX_SIZE=65536,DISABLE_CODEGEN_ROWS_THRESHOLD=0
E       Plan: 
E   ----------------
E   Max Per-Host Resource Reservation: Memory=4.88MB
E   Per-Host Resource Estimates: Memory=542.88MB
E   
E   F01:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
E   |  Per-Host Resources: mem-estimate=14.81MB mem-reservation=4.81MB runtime-filters-memory=64.00KB
E   PLAN-ROOT SINK
E   |  mem-estimate=0B mem-reservation=0B
E   |
E   05:AGGREGATE [FINALIZE]
E   |  output: count(*)
E   |  mem-estimate=10.00MB mem-reservation=0B spill-buffer=2.00MB
E   |  tuple-ids=7 row-size=8B cardinality=1
E   |
E   04:HASH JOIN [INNER JOIN, BROADCAST]
E   |  hash predicates: a.l_orderkey = -1 * l_orderkey
E   |  fk/pk conjuncts: assumed fk/pk
E   |  runtime filters: RF000[bloom] <- -1 * l_orderkey
E   |  mem-estimate=4.75MB mem-reservation=4.75MB spill-buffer=256.00KB
E   |  tuple-ids=0,4 row-size=16B cardinality=1
E   |
E   |--08:EXCHANGE [UNPARTITIONED]
E   |  |  mem-estimate=0B mem-reservation=0B
E   |  |  tuple-ids=4 row-size=8B cardinality=500000
E   |  |
E   |  F05:PLAN FRAGMENT [UNPARTITIONED] hosts=1 instances=1
E   |  Per-Host Resources: mem-estimate=0B mem-reservation=0B
E   |  07:EXCHANGE [UNPARTITIONED]
E   |  |  limit: 500000
E   |  |  mem-estimate=0B mem-reservation=0B
E   |  |  tuple-ids=4 row-size=8B cardinality=500000
E   |  |
E   |  F04:PLAN FRAGMENT [RANDOM] hosts=3 instances=3
E   |  Per-Host Resources: mem-estimate=264.00MB mem-reservation=0B
E   |  01:UNION
E   |  |  pass-through-operands: all
E   |  |  limit: 500000
E   |  |  mem-estimate=0B mem-reservation=0B
E   |  |  tuple-ids=4 row-size=8B cardinality=500000
E   |  |
E   |  |--03:SCAN HDFS [tpch.lineitem, RANDOM]
E   |  |     partitions=1/1 files=1 size=718.94MB
E   |  |     stored statistics:
E   |  |       table: rows=6001215 size=718.94MB
E   |  |       columns: all
E   |  |     extrapolated-rows=disabled
E   |  |     mem-estimate=264.00MB mem-reservation=0B
E   |  |     tuple-ids=3 row-size=8B cardinality=6001215
E   |  |
E   |  02:SCAN HDFS [tpch.lineitem, RANDOM]
E   |     partitions=1/1 files=1 size=718.94MB
E   |     stored statistics:
E   |       table: rows=6001215 size=718.94MB
E   |       columns: all
E   |     extrapolated-rows=disabled
E   |     mem-estimate=264.00MB mem-reservation=0B
E   |     tuple-ids=2 row-size=8B cardinality=6001215
E   |
E   06:EXCHANGE [UNPARTITIONED]
E   |  limit: 1
E   |  mem-estimate=0B mem-reservation=0B
E   |  tuple-ids=0 row-size=8B cardinality=1
E   |
E   F00:PLAN FRAGMENT [RANDOM] hosts=3 instances=3
E   Per-Host Resources: mem-estimate=264.06MB mem-reservation=64.00KB runtime-filters-memory=64.00KB
E   00:SCAN HDFS [tpch.lineitem a, RANDOM]
E      partitions=1/1 files=1 size=718.94MB
E      runtime filters: RF000[bloom] -> a.l_orderkey
E      stored statistics:
E        table: rows=6001215 size=718.94MB
E        columns: all
E      extrapolated-rows=disabled
E      limit: 1
E      mem-estimate=264.00MB mem-reservation=0B
E      tuple-ids=0 row-size=8B cardinality=1
E   ----------------
E       Estimated Per-Host Mem: 569245696
E       Per Host Min Reservation: ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22000(4.88 MB) ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22001(64.00 KB) ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22002(64.00 KB) 
E       Request Pool: default-pool
E       Admission result: Admitted immediately
E       Query Compilation: 4.909ms
E          - Metadata of all 1 tables cached: 544.043us (544.043us)
E          - Analysis finished: 2.901ms (2.357ms)
E          - Value transfer graph computed: 2.985ms (83.663us)
E          - Single node plan created: 3.915ms (930.418us)
E          - Runtime filters computed: 4.050ms (134.209us)
E          - Distributed plan created: 4.083ms (33.107us)
E          - Planning finished: 4.909ms (825.945us)
E       Query Timeline: 1s236ms
E          - Query submitted: 0.000ns (0.000ns)
E          - Planning finished: 7.000ms (7.000ms)
E          - Submit for admission: 7.000ms (0.000ns)
E          - Completed admission: 7.000ms (0.000ns)
E          - Ready to start on 3 backends: 7.000ms (1.000ns)
E          - All 3 execution backends (8 fragment instances) started: 8.000ms (1.000ms)
E          - First dynamic filter received: 820.015ms (812.015ms)
E          - Rows available: 1s178ms (358.006ms)
E          - First row fetched: 1s222ms (44.000ms)
E          - Last row fetched: 1s222ms (1.000ns)
E          - Released admission control resources: 1s231ms (9.000ms)
E        - ComputeScanRangeAssignmentTimer: 0.000ns
E     ImpalaServer:
E        - ClientFetchWaitTimer: 44.000ms
E        - RowMaterializationTimer: 11.000ms
E     Execution Profile 31479a87f057d480:fb4b93dc00000000:(Total: 1s176ms, non-child: 0.000ns, % non-child: 0.00%)
E       Number of filters: 1
E       Filter routing table: 
E    ID  Src. Node  Tgt. Node(s)  Target type  Partition filter  Pending (Expected)  First arrived  Completed   Enabled
E   -------------------------------------------------------------------------------------------------------------------
E     0          4             0       REMOTE             false               1 (1)            N/A        N/A      true
E       Backend startup latencies: Count: 3, min / max: 1ms / 1ms, 25th %-ile: 1ms, 50th %-ile: 1ms, 75th %-ile: 1ms, 90th %-ile: 1ms, 95th %-ile: 1ms, 99.9th %-ile: 1ms
E       Final filter table: 
E    ID  Src. Node  Tgt. Node(s)  Target type  Partition filter  Pending (Expected)  First arrived  Completed   Enabled
E   -------------------------------------------------------------------------------------------------------------------
E     0          4             0       REMOTE             false               0 (1)      820.015ms  820.015ms     false
E       Per Node Peak Memory Usage: ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22000(139.82 MB) ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22001(97.21 MB) ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22002(97.23 MB) 
E        - FiltersReceived: 1 (1)
E        - FinalizationTimer: 0.000ns
E        - NumBackends: 3 (3)
E        - NumFragmentInstances: 8 (8)
E        - NumFragments: 4 (4)
E       Averaged Fragment F01:
E         split sizes:  min: 0, max: 0, avg: 0, stddev: 0
E         completion times: min:1s224ms  max:1s224ms  mean: 1s224ms  stddev:0.000ns
E         execution rates: min:0.00 /sec  max:0.00 /sec  mean:0.00 /sec  stddev:0.00 /sec
E         num instances: 1
E       Coordinator Fragment F01:
E         Instance 31479a87f057d480:fb4b93dc00000000 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22000):
E       Averaged Fragment F00:(Total: 1s094ms, non-child: 666.680us, % non-child: 0.06%)
E         split sizes:  min: 206.94 MB, max: 256.00 MB, avg: 239.65 MB, stddev: 23.13 MB
E         completion times: min:1s029ms  max:1s223ms  mean: 1s096ms  stddev:89.628ms
E         execution rates: min:199.55 MB/sec  max:248.78 MB/sec  mean:219.22 MB/sec  stddev:21.28 MB/sec
E         num instances: 3
E          - AverageThreadTokens: 2.67 
E          - BloomFilterBytes: 0
E          - PeakMemoryUsage: 48.14 MB (50482418)
E          - PeakReservation: 64.00 KB (65536)
E          - PeakUsedReservation: 0
E          - PerHostPeakMemUsage: 111.42 MB (116832025)
E          - RowsProduced: 1 (1)
E          - TotalNetworkReceiveTime: 0.000ns
E          - TotalNetworkSendTime: 1.000ms
E          - TotalStorageWaitTime: 17.000ms
E          - TotalThreadsInvoluntaryContextSwitches: 296 (296)
E          - TotalThreadsTotalWallClockTime: 2s175ms
E            - TotalThreadsSysTime: 7.997ms
E            - TotalThreadsUserTime: 347.612ms
E          - TotalThreadsVoluntaryContextSwitches: 85 (85)
E         Buffer pool:
E            - AllocTime: 0.000ns
E            - CumulativeAllocationBytes: 0
E            - CumulativeAllocations: 0 (0)
E            - PeakReservation: 64.00 KB (65536)
E            - PeakUnpinnedBytes: 0
E            - PeakUsedReservation: 0
E            - ReadIoBytes: 0
E            - ReadIoOps: 0 (0)
E            - ReadIoWaitTime: 0.000ns
E            - ReservationLimit: 64.00 KB (65536)
E            - WriteIoBytes: 0
E            - WriteIoOps: 0 (0)
E            - WriteIoWaitTime: 0.000ns
E         Fragment Instance Lifecycle Timings:
E            - ExecTime: 955.685ms
E              - ExecTreeExecTime: 954.685ms
E            - OpenTime: 137.335ms
E              - ExecTreeOpenTime: 333.340us
E            - PrepareTime: 1.333ms
E              - ExecTreePrepareTime: 0.000ns
E         KrpcDataStreamSender (dst_id=6):(Total: 1.666ms, non-child: 1.666ms, % non-child: 100.00%)
E            - EosSent: 0 (0)
E            - NetworkThroughput: 16.05 KB/sec
E            - PeakMemoryUsage: 14.22 KB (14560)
E            - RowsSent: 0 (0)
E            - RpcFailure: 0 (0)
E            - RpcRetry: 0 (0)
E            - SerializeBatchTime: 0.000ns
E            - TotalBytesSent: 8.00 B (8)
E            - UncompressedRowBatchSize: 17.00 B (17)
E         HDFS_SCAN_NODE (id=0):(Total: 955.018ms, non-child: 955.018ms, % non-child: 100.00%)
E            - AverageHdfsReadThreadConcurrency: 0.00 
E            - AverageScannerThreadConcurrency: 2.00 
E            - BytesRead: 127.65 MB (133846845)
E            - BytesReadDataNodeCache: 0
E            - BytesReadLocal: 127.65 MB (133846845)
E            - BytesReadRemoteUnexpected: 0
E            - BytesReadShortCircuit: 127.65 MB (133846845)
E            - CachedFileHandlesHitCount: 16 (16)
E            - CachedFileHandlesMissCount: 0 (0)
E            - CollectionItemsRead: 0 (0)
E            - DecompressionTime: 0.000ns
E            - MaxCompressedTextFileLength: 0
E            - NumDisksAccessed: 1 (1)
E            - NumScannerThreadsStarted: 2 (2)
E            - PeakMemoryUsage: 48.06 MB (50392746)
E            - PerReadThreadRawHdfsThroughput: 1.68 GB/sec
E            - RemoteScanRanges: 0 (0)
E            - RowBatchQueueGetWaitTime: 166.669ms
E            - RowBatchQueuePutWaitTime: 0.000ns
E            - RowsRead: 770.00K (770000)
E            - RowsReturned: 1 (1)
E            - RowsReturnedRate: 0
E            - ScanRangesComplete: 2 (2)
E            - ScannerThreadsInvoluntaryContextSwitches: 222 (222)
E            - ScannerThreadsTotalWallClockTime: 1s082ms
E              - DelimiterParseTime: 315.339ms
E              - MaterializeTupleTime(*): 333.340us
E              - ScannerThreadsSysTime: 2.998ms
E              - ScannerThreadsUserTime: 232.297ms
E            - ScannerThreadsVoluntaryContextSwitches: 44 (44)
E            - TotalRawHdfsOpenFileTime(*): 0.000ns
E            - TotalRawHdfsReadTime(*): 73.001ms
E            - TotalReadThroughput: 16.00 MB/sec
E           Filter 0 (64.00 KB):
E              - Files processed: 0 (0)
E              - Files rejected: 0 (0)
E              - Files total: 0 (0)
E              - RowGroups processed: 0 (0)
E              - RowGroups rejected: 0 (0)
E              - RowGroups total: 0 (0)
E              - Rows processed: 0 (0)
E              - Rows rejected: 0 (0)
E              - Rows total: 0 (0)
E              - Splits processed: 0 (0)
E              - Splits rejected: 0 (0)
E              - Splits total: 0 (0)
E         CodeGen:(Total: 137.002ms, non-child: 137.002ms, % non-child: 100.00%)
E            - CodegenInvoluntaryContextSwitches: 74 (74)
E            - CodegenTotalWallClockTime: 137.002ms
E              - CodegenSysTime: 3.999ms
E              - CodegenUserTime: 113.982ms
E            - CodegenVoluntaryContextSwitches: 4 (4)
E            - CompileTime: 44.667ms
E            - IrGenerationTime: 1.666ms
E            - LoadTime: 0.000ns
E            - ModuleBitcodeSize: 2.26 MB (2373956)
E            - NumFunctions: 25 (25)
E            - NumInstructions: 421 (421)
E            - OptimizationTime: 62.667ms
E            - PeakMemoryUsage: 210.50 KB (215552)
E            - PrepareTime: 25.000ms
E       Fragment F00:
E         Instance 31479a87f057d480:fb4b93dc00000003 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22000):(Total: 1s222ms, non-child: 2.000ms, % non-child: 0.16%)
E           Hdfs split stats (<volume id>:<# splits>/<split lengths>): 0:2/256.00 MB 
E           Filter 0 arrival: 811ms
E           Fragment Instance Lifecycle Event Timeline: 1s222ms
E              - Prepare Finished: 2.000ms (2.000ms)
E              - Open Finished: 166.003ms (164.003ms)
E              - First Batch Produced: 1s222ms (1s056ms)
E              - ExecInternal Finished: 1s222ms (0.000ns)
E           MemoryUsage(500.000ms): 87.57 KB, 87.57 KB
E           ThreadUsage(500.000ms): 3, 3
E            - AverageThreadTokens: 3.00 
E            - BloomFilterBytes: 0
E            - PeakMemoryUsage: 48.13 MB (50464328)
E            - PeakReservation: 64.00 KB (65536)
E            - PeakUsedReservation: 0
E            - PerHostPeakMemUsage: 139.82 MB (146606797)
E            - RowsProduced: 1 (1)
E            - TotalNetworkReceiveTime: 0.000ns
E            - TotalNetworkSendTime: 0.000ns
E            - TotalStorageWaitTime: 17.000ms
E            - TotalThreadsInvoluntaryContextSwitches: 83 (83)
E            - TotalThreadsTotalWallClockTime: 1s220ms
E              - TotalThreadsSysTime: 7.999ms
E              - TotalThreadsUserTime: 140.978ms
E            - TotalThreadsVoluntaryContextSwitches: 45 (45)
E           Buffer pool:
E              - AllocTime: 0.000ns
E              - CumulativeAllocationBytes: 0
E              - CumulativeAllocations: 0 (0)
E              - PeakReservation: 64.00 KB (65536)
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 0
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - ReservationLimit: 64.00 KB (65536)
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Fragment Instance Lifecycle Timings:
E              - ExecTime: 1s056ms
E                - ExecTreeExecTime: 1s056ms
E              - OpenTime: 164.003ms
E                - ExecTreeOpenTime: 1.000ms
E              - PrepareTime: 2.000ms
E                - ExecTreePrepareTime: 0.000ns
E           KrpcDataStreamSender (dst_id=6):
E             BytesSent(500.000ms): 0, 0
E              - EosSent: 0 (0)
E              - NetworkThroughput: 0.00 /sec
E              - PeakMemoryUsage: 14.22 KB (14560)
E              - RowsSent: 0 (0)
E              - RpcFailure: 0 (0)
E              - RpcRetry: 0 (0)
E              - SerializeBatchTime: 0.000ns
E              - TotalBytesSent: 0
E              - UncompressedRowBatchSize: 17.00 B (17)
E           HDFS_SCAN_NODE (id=0):(Total: 1s057ms, non-child: 1s057ms, % non-child: 100.00%)
E             Hdfs split stats (<volume id>:<# splits>/<split lengths>): 0:2/256.00 MB 
E             ExecOption: TEXT Codegen Enabled, Codegen enabled: 2 out of 2
E             Runtime filters: All filters arrived. Waited 648ms
E             Hdfs Read Thread Concurrency Bucket: 0:100% 1:0% 2:0% 3:0% 4:0% 5:0% 6:0% 
E             File Formats: TEXT/NONE:1 
E             BytesRead(500.000ms): 0, 0
E              - AverageHdfsReadThreadConcurrency: 0.00 
E              - AverageScannerThreadConcurrency: 2.00 
E              - BytesRead: 80.00 MB (83886080)
E              - BytesReadDataNodeCache: 0
E              - BytesReadLocal: 80.00 MB (83886080)
E              - BytesReadRemoteUnexpected: 0
E              - BytesReadShortCircuit: 80.00 MB (83886080)
E              - CachedFileHandlesHitCount: 10 (10)
E              - CachedFileHandlesMissCount: 0 (0)
E              - CollectionItemsRead: 0 (0)
E              - DecompressionTime: 0.000ns
E              - MaxCompressedTextFileLength: 0
E              - NumDisksAccessed: 1 (1)
E              - NumScannerThreadsStarted: 2 (2)
E              - PeakMemoryUsage: 48.04 MB (50374656)
E              - PerReadThreadRawHdfsThroughput: 1.45 GB/sec
E              - RemoteScanRanges: 0 (0)
E              - RowBatchQueueGetWaitTime: 85.001ms
E              - RowBatchQueuePutWaitTime: 0.000ns
E              - RowsRead: 356.56K (356558)
E              - RowsReturned: 1 (1)
E              - RowsReturnedRate: 0
E              - ScanRangesComplete: 2 (2)
E              - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E              - ScannerThreadsTotalWallClockTime: 0.000ns
E                - DelimiterParseTime: 153.002ms
E                - MaterializeTupleTime(*): 0.000ns
E                - ScannerThreadsSysTime: 0.000ns
E                - ScannerThreadsUserTime: 0.000ns
E              - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E              - TotalRawHdfsOpenFileTime(*): 0.000ns
E              - TotalRawHdfsReadTime(*): 54.001ms
E              - TotalReadThroughput: 0.00 /sec
E             Filter 0 (64.00 KB):
E                - Files processed: 0 (0)
E                - Files rejected: 0 (0)
E                - Files total: 0 (0)
E                - RowGroups processed: 0 (0)
E                - RowGroups rejected: 0 (0)
E                - RowGroups total: 0 (0)
E                - Rows processed: 0 (0)
E                - Rows rejected: 0 (0)
E                - Rows total: 0 (0)
E                - Splits processed: 0 (0)
E                - Splits rejected: 0 (0)
E                - Splits total: 0 (0)
E           CodeGen:(Total: 163.003ms, non-child: 163.003ms, % non-child: 100.00%)
E              - CodegenInvoluntaryContextSwitches: 83 (83)
E              - CodegenTotalWallClockTime: 163.003ms
E                - CodegenSysTime: 7.999ms
E                - CodegenUserTime: 138.978ms
E              - CodegenVoluntaryContextSwitches: 10 (10)
E              - CompileTime: 51.000ms
E              - IrGenerationTime: 1.000ms
E              - LoadTime: 0.000ns
E              - ModuleBitcodeSize: 2.26 MB (2373956)
E              - NumFunctions: 25 (25)
E              - NumInstructions: 421 (421)
E              - OptimizationTime: 80.001ms
E              - PeakMemoryUsage: 210.50 KB (215552)
E              - PrepareTime: 26.000ms
E         Instance 31479a87f057d480:fb4b93dc00000001 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22002):(Total: 1s035ms, non-child: 0.000ns, % non-child: 0.00%)
E           Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/78.94 MB 0:1/128.00 MB 
E           Filter 0 arrival: 812ms
E           Fragment Instance Lifecycle Event Timeline: 1s035ms
E              - Prepare Finished: 2.000ms (2.000ms)
E              - Open Finished: 118.002ms (116.002ms)
E              - First Batch Produced: 1s033ms (915.017ms)
E              - First Batch Sent: 1s033ms (0.000ns)
E              - ExecInternal Finished: 1s035ms (2.000ms)
E           MemoryUsage(500.000ms): 87.57 KB, 48.12 MB
E           ThreadUsage(500.000ms): 3, 3
E            - AverageThreadTokens: 3.00 
E            - BloomFilterBytes: 0
E            - PeakMemoryUsage: 48.16 MB (50500168)
E            - PeakReservation: 64.00 KB (65536)
E            - PeakUsedReservation: 0
E            - PerHostPeakMemUsage: 97.23 MB (101953344)
E            - RowsProduced: 1 (1)
E            - TotalNetworkReceiveTime: 0.000ns
E            - TotalNetworkSendTime: 2.000ms
E            - TotalStorageWaitTime: 20.000ms
E            - TotalThreadsInvoluntaryContextSwitches: 365 (365)
E            - TotalThreadsTotalWallClockTime: 2s660ms
E              - TotalThreadsSysTime: 6.997ms
E              - TotalThreadsUserTime: 453.929ms
E            - TotalThreadsVoluntaryContextSwitches: 105 (105)
E           Buffer pool:
E              - AllocTime: 0.000ns
E              - CumulativeAllocationBytes: 0
E              - CumulativeAllocations: 0 (0)
E              - PeakReservation: 64.00 KB (65536)
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 0
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - ReservationLimit: 64.00 KB (65536)
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Fragment Instance Lifecycle Timings:
E              - ExecTime: 917.017ms
E                - ExecTreeExecTime: 915.017ms
E              - OpenTime: 116.002ms
E                - ExecTreeOpenTime: 0.000ns
E              - PrepareTime: 2.000ms
E                - ExecTreePrepareTime: 0.000ns
E           KrpcDataStreamSender (dst_id=6):(Total: 4.000ms, non-child: 2.000ms, % non-child: 50.00%)
E             BytesSent(500.000ms): 0, 0
E              - EosSent: 1 (1)
E              - NetworkThroughput: 20.31 KB/sec
E              - PeakMemoryUsage: 14.22 KB (14560)
E              - RowsSent: 1 (1)
E              - RpcFailure: 0 (0)
E              - RpcRetry: 0 (0)
E              - SerializeBatchTime: 0.000ns
E              - TotalBytesSent: 13.00 B (13)
E              - UncompressedRowBatchSize: 17.00 B (17)
E           HDFS_SCAN_NODE (id=0):(Total: 915.017ms, non-child: 915.017ms, % non-child: 100.00%)
E             Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/78.94 MB 0:1/128.00 MB 
E             ExecOption: TEXT Codegen Enabled, Codegen enabled: 2 out of 2
E             Runtime filters: All filters arrived. Waited 708ms
E             Hdfs Read Thread Concurrency Bucket: 0:100% 1:0% 2:0% 3:0% 4:0% 5:0% 6:0% 
E             BytesRead(500.000ms): 0, 48.00 MB
E              - AverageHdfsReadThreadConcurrency: 0.00 
E              - AverageScannerThreadConcurrency: 2.00 
E              - BytesRead: 150.94 MB (158270904)
E              - BytesReadDataNodeCache: 0
E              - BytesReadLocal: 150.94 MB (158270904)
E              - BytesReadRemoteUnexpected: 0
E              - BytesReadShortCircuit: 150.94 MB (158270904)
E              - CachedFileHandlesHitCount: 19 (19)
E              - CachedFileHandlesMissCount: 0 (0)
E              - CollectionItemsRead: 0 (0)
E              - DecompressionTime: 0.000ns
E              - MaxCompressedTextFileLength: 0
E              - NumDisksAccessed: 2 (2)
E              - NumScannerThreadsStarted: 2 (2)
E              - PeakMemoryUsage: 48.08 MB (50410496)
E              - PerReadThreadRawHdfsThroughput: 1.75 GB/sec
E              - RemoteScanRanges: 0 (0)
E              - RowBatchQueueGetWaitTime: 208.004ms
E              - RowBatchQueuePutWaitTime: 0.000ns
E              - RowsRead: 967.94K (967942)
E              - RowsReturned: 1 (1)
E              - RowsReturnedRate: 1.00 /sec
E              - ScanRangesComplete: 2 (2)
E              - ScannerThreadsInvoluntaryContextSwitches: 303 (303)
E              - ScannerThreadsTotalWallClockTime: 1s627ms
E                - DelimiterParseTime: 395.007ms
E                - MaterializeTupleTime(*): 0.000ns
E                - ScannerThreadsSysTime: 2.998ms
E                - ScannerThreadsUserTime: 352.945ms
E              - ScannerThreadsVoluntaryContextSwitches: 66 (66)
E              - TotalRawHdfsOpenFileTime(*): 0.000ns
E              - TotalRawHdfsReadTime(*): 84.001ms
E              - TotalReadThroughput: 48.00 MB/sec
E             Filter 0 (64.00 KB):
E                - Files processed: 0 (0)
E                - Files rejected: 0 (0)
E                - Files total: 0 (0)
E                - RowGroups processed: 0 (0)
E                - RowGroups rejected: 0 (0)
E                - RowGroups total: 0 (0)
E                - Rows processed: 0 (0)
E                - Rows rejected: 0 (0)
E                - Rows total: 0 (0)
E                - Splits processed: 0 (0)
E                - Splits rejected: 0 (0)
E                - Splits total: 0 (0)
E           CodeGen:(Total: 116.002ms, non-child: 116.002ms, % non-child: 100.00%)
E              - CodegenInvoluntaryContextSwitches: 62 (62)
E              - CodegenTotalWallClockTime: 116.002ms
E                - CodegenSysTime: 1.999ms
E                - CodegenUserTime: 99.984ms
E              - CodegenVoluntaryContextSwitches: 1 (1)
E              - CompileTime: 32.000ms
E              - IrGenerationTime: 2.000ms
E              - LoadTime: 0.000ns
E              - ModuleBitcodeSize: 2.26 MB (2373956)
E              - NumFunctions: 25 (25)
E              - NumInstructions: 421 (421)
E              - OptimizationTime: 53.001ms
E              - PeakMemoryUsage: 210.50 KB (215552)
E              - PrepareTime: 28.000ms
E         Instance 31479a87f057d480:fb4b93dc00000002 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22001):(Total: 1s026ms, non-child: 0.000ns, % non-child: 0.00%)
E           Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/128.00 MB 0:1/128.00 MB 
E           Filter 0 arrival: 812ms
E           Fragment Instance Lifecycle Event Timeline: 1s028ms
E              - Prepare Finished: 2.000ms (2.000ms)
E              - Open Finished: 134.002ms (132.002ms)
E              - First Batch Produced: 1s027ms (893.017ms)
E              - First Batch Sent: 1s027ms (0.000ns)
E              - ExecInternal Finished: 1s028ms (1.000ms)
E           MemoryUsage(500.000ms): 296.72 KB, 87.57 KB
E           ThreadUsage(500.000ms): 1, 3
E            - AverageThreadTokens: 2.00 
E            - BloomFilterBytes: 0
E            - PeakMemoryUsage: 48.14 MB (50482760)
E            - PeakReservation: 64.00 KB (65536)
E            - PeakUsedReservation: 0
E            - PerHostPeakMemUsage: 97.21 MB (101935936)
E            - RowsProduced: 1 (1)
E            - TotalNetworkReceiveTime: 0.000ns
E            - TotalNetworkSendTime: 1.000ms
E            - TotalStorageWaitTime: 14.000ms
E            - TotalThreadsInvoluntaryContextSwitches: 442 (442)
E            - TotalThreadsTotalWallClockTime: 2s645ms
E              - TotalThreadsSysTime: 8.997ms
E              - TotalThreadsUserTime: 447.931ms
E            - TotalThreadsVoluntaryContextSwitches: 105 (105)
E           Buffer pool:
E              - AllocTime: 0.000ns
E              - CumulativeAllocationBytes: 0
E              - CumulativeAllocations: 0 (0)
E              - PeakReservation: 64.00 KB (65536)
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 0
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - ReservationLimit: 64.00 KB (65536)
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Fragment Instance Lifecycle Timings:
E              - ExecTime: 894.017ms
E                - ExecTreeExecTime: 893.017ms
E              - OpenTime: 132.002ms
E                - ExecTreeOpenTime: 0.000ns
E              - PrepareTime: 0.000ns
E                - ExecTreePrepareTime: 0.000ns
E           KrpcDataStreamSender (dst_id=6):(Total: 1.000ms, non-child: 0.000ns, % non-child: 0.00%)
E             BytesSent(500.000ms): 0, 0
E              - EosSent: 1 (1)
E              - NetworkThroughput: 27.83 KB/sec
E              - PeakMemoryUsage: 14.22 KB (14560)
E              - RowsSent: 1 (1)
E              - RpcFailure: 0 (0)
E              - RpcRetry: 0 (0)
E              - SerializeBatchTime: 0.000ns
E              - TotalBytesSent: 13.00 B (13)
E              - UncompressedRowBatchSize: 17.00 B (17)
E           HDFS_SCAN_NODE (id=0):(Total: 893.017ms, non-child: 893.017ms, % non-child: 100.00%)
E             Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/128.00 MB 0:1/128.00 MB 
E             ExecOption: TEXT Codegen Enabled, Codegen enabled: 2 out of 2
E             Runtime filters: All filters arrived. Waited 687ms
E             Hdfs Read Thread Concurrency Bucket: 0:100% 1:0% 2:0% 3:0% 4:0% 5:0% 6:0% 
E             BytesRead(500.000ms): 0, 0
E              - AverageHdfsReadThreadConcurrency: 0.00 
E              - AverageScannerThreadConcurrency: 2.00 
E              - BytesRead: 152.00 MB (159383552)
E              - BytesReadDataNodeCache: 0
E              - BytesReadLocal: 152.00 MB (159383552)
E              - BytesReadRemoteUnexpected: 0
E              - BytesReadShortCircuit: 152.00 MB (159383552)
E              - CachedFileHandlesHitCount: 19 (19)
E              - CachedFileHandlesMissCount: 0 (0)
E              - CollectionItemsRead: 0 (0)
E              - DecompressionTime: 0.000ns
E              - MaxCompressedTextFileLength: 0
E              - NumDisksAccessed: 2 (2)
E              - NumScannerThreadsStarted: 2 (2)
E              - PeakMemoryUsage: 48.06 MB (50393088)
E              - PerReadThreadRawHdfsThroughput: 1.83 GB/sec
E              - RemoteScanRanges: 0 (0)
E              - RowBatchQueueGetWaitTime: 207.004ms
E              - RowBatchQueuePutWaitTime: 0.000ns
E              - RowsRead: 985.50K (985501)
E              - RowsReturned: 1 (1)
E              - RowsReturnedRate: 1.00 /sec
E              - ScanRangesComplete: 2 (2)
E              - ScannerThreadsInvoluntaryContextSwitches: 363 (363)
E              - ScannerThreadsTotalWallClockTime: 1s619ms
E                - DelimiterParseTime: 398.007ms
E                - MaterializeTupleTime(*): 1.000ms
E                - ScannerThreadsSysTime: 5.998ms
E                - ScannerThreadsUserTime: 343.947ms
E              - ScannerThreadsVoluntaryContextSwitches: 66 (66)
E              - TotalRawHdfsOpenFileTime(*): 0.000ns
E              - TotalRawHdfsReadTime(*): 81.001ms
E              - TotalReadThroughput: 0.00 /sec
E             Filter 0 (64.00 KB):
E                - Files processed: 0 (0)
E                - Files rejected: 0 (0)
E                - Files total: 0 (0)
E                - RowGroups processed: 0 (0)
E                - RowGroups rejected: 0 (0)
E                - RowGroups total: 0 (0)
E                - Rows processed: 0 (0)
E                - Rows rejected: 0 (0)
E                - Rows total: 0 (0)
E                - Splits processed: 0 (0)
E                - Splits rejected: 0 (0)
E                - Splits total: 0 (0)
E           CodeGen:(Total: 132.002ms, non-child: 132.002ms, % non-child: 100.00%)
E              - CodegenInvoluntaryContextSwitches: 79 (79)
E              - CodegenTotalWallClockTime: 132.002ms
E                - CodegenSysTime: 1.999ms
E                - CodegenUserTime: 102.984ms
E              - CodegenVoluntaryContextSwitches: 2 (2)
E              - CompileTime: 51.000ms
E              - IrGenerationTime: 2.000ms
E              - LoadTime: 0.000ns
E              - ModuleBitcodeSize: 2.26 MB (2373956)
E              - NumFunctions: 25 (25)
E              - NumInstructions: 421 (421)
E              - OptimizationTime: 55.001ms
E              - PeakMemoryUsage: 210.50 KB (215552)
E              - PrepareTime: 21.000ms
E       Averaged Fragment F05:(Total: 806.015ms, non-child: 15.000ms, % non-child: 1.86%)
E         split sizes:  min: 0, max: 0, avg: 0, stddev: 0
E         completion times: min:1s139ms  max:1s139ms  mean: 1s139ms  stddev:0.000ns
E         execution rates: min:0.00 /sec  max:0.00 /sec  mean:0.00 /sec  stddev:0.00 /sec
E         num instances: 1
E          - AverageThreadTokens: 1.00 
E          - BloomFilterBytes: 0
E          - PeakMemoryUsage: 14.19 MB (14876435)
E          - PeakReservation: 0
E          - PeakUsedReservation: 0
E          - PerHostPeakMemUsage: 98.00 MB (102757711)
E          - RowsProduced: 500.00K (500000)
E          - TotalNetworkReceiveTime: 136.002ms
E          - TotalNetworkSendTime: 552.010ms
E          - TotalStorageWaitTime: 0.000ns
E          - TotalThreadsInvoluntaryContextSwitches: 54 (54)
E          - TotalThreadsTotalWallClockTime: 805.015ms
E            - TotalThreadsSysTime: 32.994ms
E            - TotalThreadsUserTime: 72.988ms
E          - TotalThreadsVoluntaryContextSwitches: 990 (990)
E         Buffer pool:
E            - AllocTime: 0.000ns
E            - CumulativeAllocationBytes: 0
E            - CumulativeAllocations: 0 (0)
E            - PeakReservation: 0
E            - PeakUnpinnedBytes: 0
E            - PeakUsedReservation: 0
E            - ReadIoBytes: 0
E            - ReadIoOps: 0 (0)
E            - ReadIoWaitTime: 0.000ns
E            - ReservationLimit: 0
E            - WriteIoBytes: 0
E            - WriteIoOps: 0 (0)
E            - WriteIoWaitTime: 0.000ns
E         Fragment Instance Lifecycle Timings:
E            - ExecTime: 635.012ms
E              - ExecTreeExecTime: 39.000ms
E            - OpenTime: 170.003ms
E              - ExecTreeOpenTime: 110.002ms
E            - PrepareTime: 1.000ms
E              - ExecTreePrepareTime: 0.000ns
E         KrpcDataStreamSender (dst_id=8):(Total: 583.011ms, non-child: 583.011ms, % non-child: 100.00%)
E            - EosSent: 1 (1)
E            - NetworkThroughput: 6.13 MB/sec
E            - PeakMemoryUsage: 14.22 KB (14560)
E            - RowsSent: 500.00K (500000)
E            - RpcFailure: 0 (0)
E            - RpcRetry: 0 (0)
E            - SerializeBatchTime: 17.000ms
E            - TotalBytesSent: 2.73 MB (2858097)
E            - UncompressedRowBatchSize: 8.11 MB (8500000)
E         EXCHANGE_NODE (id=7):(Total: 148.002ms, non-child: 148.002ms, % non-child: 100.00%)
E            - ConvertRowBatchTime: 10.000ms
E            - PeakMemoryUsage: 14.17 MB (14853683)
E            - RowsReturned: 500.00K (500000)
E            - RowsReturnedRate: 3.38 M/sec
E           Buffer pool:
E              - AllocTime: 74.001ms
E              - CumulativeAllocationBytes: 25.57 MB (26812416)
E              - CumulativeAllocations: 2.18K (2182)
E              - PeakReservation: 14.15 MB (14835712)
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 14.15 MB (14835712)
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Dequeue:
E              - FirstBatchWaitTime: 110.002ms
E              - TotalBytesDequeued: 8.12 MB (8512512)
E              - TotalGetBatchTime: 135.002ms
E                - DataWaitTime: 122.002ms
E           Enqueue:
E              - DeserializeRowBatchTime: 84.001ms
E              - TotalBatchesEnqueued: 1.09K (1091)
E              - TotalBatchesReceived: 1.09K (1094)
E              - TotalBytesReceived: 6.11 MB (6403771)
E              - TotalEarlySenders: 0 (0)
E              - TotalEosReceived: 0 (0)
E              - TotalHasDeferredRPCsTime: 107.215ms
E              - TotalRPCsDeferred: 122 (122)
E         CodeGen:(Total: 60.001ms, non-child: 60.001ms, % non-child: 100.00%)
E            - CodegenInvoluntaryContextSwitches: 36 (36)
E            - CodegenTotalWallClockTime: 60.001ms
E              - CodegenSysTime: 2.999ms
E              - CodegenUserTime: 19.996ms
E            - CodegenVoluntaryContextSwitches: 11 (11)
E            - CompileTime: 0.000ns
E            - IrGenerationTime: 1.000ms
E            - LoadTime: 0.000ns
E            - ModuleBitcodeSize: 2.26 MB (2373956)
E            - NumFunctions: 0 (0)
E            - NumInstructions: 0 (0)
E            - OptimizationTime: 0.000ns
E            - PeakMemoryUsage: 0
E            - PrepareTime: 59.001ms
E       Fragment F05:
E         Instance 31479a87f057d480:fb4b93dc00000007 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22000):(Total: 806.015ms, non-child: 15.000ms, % non-child: 1.86%)
E           Fragment Instance Lifecycle Event Timeline: 812.015ms
E              - Prepare Finished: 7.000ms (7.000ms)
E              - Open Finished: 177.003ms (170.003ms)
E              - First Batch Produced: 177.003ms (0.000ns)
E              - First Batch Sent: 178.003ms (1.000ms)
E              - ExecInternal Finished: 812.015ms (634.012ms)
E           MemoryUsage(500.000ms): 2.73 MB, 14.18 MB
E           ThreadUsage(500.000ms): 1, 1
E            - AverageThreadTokens: 1.00 
E            - BloomFilterBytes: 0
E            - PeakMemoryUsage: 14.19 MB (14876435)
E            - PeakReservation: 0
E            - PeakUsedReservation: 0
E            - PerHostPeakMemUsage: 98.00 MB (102757711)
E            - RowsProduced: 500.00K (500000)
E            - TotalNetworkReceiveTime: 136.002ms
E            - TotalNetworkSendTime: 552.010ms
E            - TotalStorageWaitTime: 0.000ns
E            - TotalThreadsInvoluntaryContextSwitches: 54 (54)
E            - TotalThreadsTotalWallClockTime: 805.015ms
E              - TotalThreadsSysTime: 32.994ms
E              - TotalThreadsUserTime: 72.988ms
E            - TotalThreadsVoluntaryContextSwitches: 990 (990)
E           Buffer pool:
E              - AllocTime: 0.000ns
E              - CumulativeAllocationBytes: 0
E              - CumulativeAllocations: 0 (0)
E              - PeakReservation: 0
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 0
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - ReservationLimit: 0
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Fragment Instance Lifecycle Timings:
E              - ExecTime: 635.012ms
E                - ExecTreeExecTime: 39.000ms
E              - OpenTime: 170.003ms
E                - ExecTreeOpenTime: 110.002ms
E              - PrepareTime: 1.000ms
E                - ExecTreePrepareTime: 0.000ns
E           KrpcDataStreamSender (dst_id=8):(Total: 583.011ms, non-child: 31.000ms, % non-child: 5.32%)
E             BytesSent(500.000ms): 714.69 KB, 2.72 MB
E              - EosSent: 1 (1)
E              - NetworkThroughput: 6.13 MB/sec
E              - PeakMemoryUsage: 14.22 KB (14560)
E              - RowsSent: 500.00K (500000)
E              - RpcFailure: 0 (0)
E              - RpcRetry: 0 (0)
E              - SerializeBatchTime: 17.000ms
E              - TotalBytesSent: 2.73 MB (2858097)
E              - UncompressedRowBatchSize: 8.11 MB (8500000)
E           EXCHANGE_NODE (id=7):(Total: 148.002ms, non-child: 26.000ms, % non-child: 17.57%)
E              - ConvertRowBatchTime: 10.000ms
E              - PeakMemoryUsage: 14.17 MB (14853683)
E              - RowsReturned: 500.00K (500000)
E              - RowsReturnedRate: 3.38 M/sec
E             Buffer pool:
E                - AllocTime: 74.001ms
E                - CumulativeAllocationBytes: 25.57 MB (26812416)
E                - CumulativeAllocations: 2.18K (2182)
E                - PeakReservation: 14.15 MB (14835712)
E                - PeakUnpinnedBytes: 0
E                - PeakUsedReservation: 14.15 MB (14835712)
E                - ReadIoBytes: 0
E                - ReadIoOps: 0 (0)
E                - ReadIoWaitTime: 0.000ns
E                - WriteIoBytes: 0
E                - WriteIoOps: 0 (0)
E                - WriteIoWaitTime: 0.000ns
E             Dequeue:
E               BytesDequeued(500.000ms): 2.09 MB, 8.12 MB
E                - FirstBatchWaitTime: 110.002ms
E                - TotalBytesDequeued: 8.12 MB (8512512)
E                - TotalGetBatchTime: 135.002ms
E                  - DataWaitTime: 122.002ms
E             Enqueue:
E               BytesReceived(500.000ms): 1.33 MB, 6.10 MB
E               DeferredQueueSize(500.000ms): 0, 2
E                - DispatchTime: (Avg: 116.175us ; Min: 42.951us ; Max: 4.232ms ; Number of samples: 1094)
E                - DeserializeRowBatchTime: 84.001ms
E                - TotalBatchesEnqueued: 1.09K (1091)
E                - TotalBatchesReceived: 1.09K (1094)
E                - TotalBytesReceived: 6.11 MB (6403771)
E                - TotalEarlySenders: 0 (0)
E                - TotalEosReceived: 0 (0)
E                - TotalHasDeferredRPCsTime: 107.215ms
E                - TotalRPCsDeferred: 122 (122)
E           CodeGen:(Total: 60.001ms, non-child: 60.001ms, % non-child: 100.00%)
E              - CodegenInvoluntaryContextSwitches: 36 (36)
E              - CodegenTotalWallClockTime: 60.001ms
E                - CodegenSysTime: 2.999ms
E                - CodegenUserTime: 19.996ms
E              - CodegenVoluntaryContextSwitches: 11 (11)
E              - CompileTime: 0.000ns
E              - IrGenerationTime: 1.000ms
E              - LoadTime: 0.000ns
E              - ModuleBitcodeSize: 2.26 MB (2373956)
E              - NumFunctions: 0 (0)
E              - NumInstructions: 0 (0)
E              - OptimizationTime: 0.000ns
E              - PeakMemoryUsage: 0
E              - PrepareTime: 59.001ms
E       Averaged Fragment F04:(Total: 1s163ms, non-child: 5.333ms, % non-child: 0.46%)
E         split sizes:  min: 413.88 MB, max: 512.00 MB, avg: 479.29 MB, stddev: 46.26 MB
E         completion times: min:1s159ms  max:1s221ms  mean: 1s182ms  stddev:27.354ms
E         execution rates: min:357.09 MB/sec  max:438.35 MB/sec  mean:404.92 MB/sec  stddev:34.70 MB/sec
E         num instances: 3
E          - AverageThreadTokens: 2.78 
E          - BloomFilterBytes: 0
E          - PeakMemoryUsage: 49.09 MB (51470925)
E          - PeakReservation: 0
E          - PeakUsedReservation: 0
E          - PerHostPeakMemUsage: 111.42 MB (116832025)
E          - RowsProduced: 500.00K (500000)
E          - TotalNetworkReceiveTime: 0.000ns
E          - TotalNetworkSendTime: 858.016ms
E          - TotalStorageWaitTime: 7.666ms
E          - TotalThreadsInvoluntaryContextSwitches: 197 (197)
E          - TotalThreadsTotalWallClockTime: 1s159ms
E            - TotalThreadsSysTime: 32.328ms
E            - TotalThreadsUserTime: 181.972ms
E          - TotalThreadsVoluntaryContextSwitches: 864 (864)
E         Buffer pool:
E            - AllocTime: 0.000ns
E            - CumulativeAllocationBytes: 0
E            - CumulativeAllocations: 0 (0)
E            - PeakReservation: 0
E            - PeakUnpinnedBytes: 0
E            - PeakUsedReservation: 0
E            - ReadIoBytes: 0
E            - ReadIoOps: 0 (0)
E            - ReadIoWaitTime: 0.000ns
E            - ReservationLimit: 0
E            - WriteIoBytes: 0
E            - WriteIoOps: 0 (0)
E            - WriteIoWaitTime: 0.000ns
E         Fragment Instance Lifecycle Timings:
E            - ExecTime: 956.685ms
E              - ExecTreeExecTime: 62.667ms
E            - OpenTime: 203.003ms
E              - ExecTreeOpenTime: 0.000ns
E            - PrepareTime: 4.000ms
E              - ExecTreePrepareTime: 333.340us
E         KrpcDataStreamSender (dst_id=7):(Total: 893.017ms, non-child: 893.017ms, % non-child: 100.00%)
E            - EosSent: 0 (0)
E            - NetworkThroughput: 5.93 MB/sec
E            - PeakMemoryUsage: 14.22 KB (14560)
E            - RowsSent: 500.00K (500000)
E            - RpcFailure: 0 (0)
E            - RpcRetry: 0 (0)
E            - SerializeBatchTime: 24.000ms
E            - TotalBytesSent: 2.04 MB (2140461)
E            - UncompressedRowBatchSize: 8.11 MB (8500000)
E         UNION_NODE (id=1):(Total: 62.667ms, non-child: 0.000ns, % non-child: 0.00%)
E            - PeakMemoryUsage: 0
E            - RowsReturned: 500.00K (500000)
E            - RowsReturnedRate: 10.32 M/sec
E           HDFS_SCAN_NODE (id=3):
E              - BytesRead: 0
E              - CollectionItemsRead: 0 (0)
E              - PeakMemoryUsage: 0
E              - RowBatchQueueGetWaitTime: 0.000ns
E              - RowBatchQueuePutWaitTime: 0.000ns
E              - RowsRead: 0 (0)
E              - RowsReturned: 0 (0)
E              - RowsReturnedRate: 0
E              - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E              - ScannerThreadsTotalWallClockTime: 0.000ns
E                - MaterializeTupleTime(*): 0.000ns
E                - ScannerThreadsSysTime: 0.000ns
E                - ScannerThreadsUserTime: 0.000ns
E              - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E              - TotalReadThroughput: 0.00 /sec
E         HDFS_SCAN_NODE (id=2):(Total: 63.001ms, non-child: 63.001ms, % non-child: 100.00%)
E            - AverageHdfsReadThreadConcurrency: 0.33 
E            - AverageScannerThreadConcurrency: 2.00 
E            - BytesRead: 98.67 MB (103459498)
E            - BytesReadDataNodeCache: 0
E            - BytesReadLocal: 0
E            - BytesReadRemoteUnexpected: 0
E            - BytesReadShortCircuit: 0
E            - CachedFileHandlesHitCount: 0 (0)
E            - CachedFileHandlesMissCount: 0 (0)
E            - CollectionItemsRead: 0 (0)
E            - DecompressionTime: 0.000ns
E            - MaxCompressedTextFileLength: 0
E            - NumDisksAccessed: 0 (0)
E            - NumScannerThreadsStarted: 2 (2)
E            - PeakMemoryUsage: 49.06 MB (51443029)
E            - PerReadThreadRawHdfsThroughput: 1.67 GB/sec
E            - RemoteScanRanges: 0 (0)
E            - RowBatchQueueGetWaitTime: 0.000ns
E            - RowBatchQueuePutWaitTime: 0.000ns
E            - RowsRead: 535.55K (535552)
E            - RowsReturned: 500.74K (500736)
E            - RowsReturnedRate: 10.29 M/sec
E            - ScanRangesComplete: 0 (0)
E            - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E            - ScannerThreadsTotalWallClockTime: 0.000ns
E              - DelimiterParseTime: 243.671ms
E              - MaterializeTupleTime(*): 65.001ms
E              - ScannerThreadsSysTime: 0.000ns
E              - ScannerThreadsUserTime: 0.000ns
E            - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E            - TotalRawHdfsOpenFileTime(*): 333.340us
E            - TotalRawHdfsReadTime(*): 58.667ms
E            - TotalReadThroughput: 80.00 MB/sec
E         CodeGen:(Total: 202.670ms, non-child: 202.670ms, % non-child: 100.00%)
E            - CodegenInvoluntaryContextSwitches: 188 (188)
E            - CodegenTotalWallClockTime: 202.670ms
E              - CodegenSysTime: 7.331ms
E              - CodegenUserTime: 135.646ms
E            - CodegenVoluntaryContextSwitches: 7 (7)
E            - CompileTime: 51.000ms
E            - IrGenerationTime: 3.666ms
E            - LoadTime: 0.000ns
E            - ModuleBitcodeSize: 2.26 MB (2373956)
E            - NumFunctions: 30 (30)
E            - NumInstructions: 562 (562)
E            - OptimizationTime: 113.668ms
E            - PeakMemoryUsage: 281.00 KB (287744)
E            - PrepareTime: 33.667ms
E       Fragment F04:
E         Instance 31479a87f057d480:fb4b93dc00000004 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22000):(Total: 1s185ms, non-child: 12.000ms, % non-child: 1.01%)
E           Hdfs split stats (<volume id>:<# splits>/<split lengths>): 0:4/512.00 MB 
E           Fragment Instance Lifecycle Event Timeline: 1s205ms
E              - Prepare Finished: 33.000ms (33.000ms)
E              - Open Finished: 298.005ms (265.005ms)
E              - First Batch Produced: 306.005ms (8.000ms)
E              - First Batch Sent: 306.005ms (0.000ns)
E              - ExecInternal Finished: 1s205ms (899.017ms)
E           MemoryUsage(500.000ms): 40.31 MB, 49.07 MB
E           ThreadUsage(500.000ms): 3, 3
E            - AverageThreadTokens: 3.00 
E            - BloomFilterBytes: 0
E            - PeakMemoryUsage: 49.08 MB (51461368)
E            - PeakReservation: 0
E            - PeakUsedReservation: 0
E            - PerHostPeakMemUsage: 139.82 MB (146606797)
E            - RowsProduced: 500.00K (500000)
E            - TotalNetworkReceiveTime: 0.000ns
E            - TotalNetworkSendTime: 817.015ms
E            - TotalStorageWaitTime: 13.000ms
E            - TotalThreadsInvoluntaryContextSwitches: 233 (233)
E            - TotalThreadsTotalWallClockTime: 1s173ms
E              - TotalThreadsSysTime: 25.996ms
E              - TotalThreadsUserTime: 184.972ms
E            - TotalThreadsVoluntaryContextSwitches: 691 (691)
E           Buffer pool:
E              - AllocTime: 0.000ns
E              - CumulativeAllocationBytes: 0
E              - CumulativeAllocations: 0 (0)
E              - PeakReservation: 0
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 0
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - ReservationLimit: 0
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Fragment Instance Lifecycle Timings:
E              - ExecTime: 907.017ms
E                - ExecTreeExecTime: 62.001ms
E              - OpenTime: 266.005ms
E                - ExecTreeOpenTime: 0.000ns
E              - PrepareTime: 12.000ms
E                - ExecTreePrepareTime: 1.000ms
E           KrpcDataStreamSender (dst_id=7):(Total: 845.016ms, non-child: 28.000ms, % non-child: 3.31%)
E             BytesSent(500.000ms): 17.10 KB, 1.49 MB
E              - EosSent: 0 (0)
E              - NetworkThroughput: 4.79 MB/sec
E              - PeakMemoryUsage: 14.22 KB (14560)
E              - RowsSent: 500.00K (500000)
E              - RpcFailure: 0 (0)
E              - RpcRetry: 0 (0)
E              - SerializeBatchTime: 20.000ms
E              - TotalBytesSent: 1.50 MB (1574159)
E              - UncompressedRowBatchSize: 8.11 MB (8500000)
E           UNION_NODE (id=1):(Total: 63.001ms, non-child: 0.000ns, % non-child: 0.00%)
E             ExecOption: Codegen Enabled
E              - PeakMemoryUsage: 0
E              - RowsReturned: 500.00K (500000)
E              - RowsReturnedRate: 7.94 M/sec
E             HDFS_SCAN_NODE (id=3):
E               Hdfs split stats (<volume id>:<# splits>/<split lengths>): 0:2/256.00 MB 
E               ExecOption: TEXT Codegen Enabled
E               BytesRead(500.000ms): 0, 0
E                - BytesRead: 0
E                - CollectionItemsRead: 0 (0)
E                - PeakMemoryUsage: 0
E                - RowBatchQueueGetWaitTime: 0.000ns
E                - RowBatchQueuePutWaitTime: 0.000ns
E                - RowsRead: 0 (0)
E                - RowsReturned: 0 (0)
E                - RowsReturnedRate: 0
E                - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E                - ScannerThreadsTotalWallClockTime: 0.000ns
E                  - MaterializeTupleTime(*): 0.000ns
E                  - ScannerThreadsSysTime: 0.000ns
E                  - ScannerThreadsUserTime: 0.000ns
E                - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E                - TotalReadThroughput: 0.00 /sec
E           HDFS_SCAN_NODE (id=2):(Total: 64.001ms, non-child: 64.001ms, % non-child: 100.00%)
E             Hdfs split stats (<volume id>:<# splits>/<split lengths>): 0:2/256.00 MB 
E             ExecOption: TEXT Codegen Enabled
E             BytesRead(500.000ms): 24.00 MB, 80.00 MB
E              - AverageHdfsReadThreadConcurrency: 1.00 
E              - AverageScannerThreadConcurrency: 2.00 
E              - BytesRead: 104.00 MB (109051904)
E              - BytesReadDataNodeCache: 0
E              - BytesReadLocal: 0
E              - BytesReadRemoteUnexpected: 0
E              - BytesReadShortCircuit: 0
E              - CachedFileHandlesHitCount: 0 (0)
E              - CachedFileHandlesMissCount: 0 (0)
E              - CollectionItemsRead: 0 (0)
E              - DecompressionTime: 0.000ns
E              - MaxCompressedTextFileLength: 0
E              - NumDisksAccessed: 0 (0)
E              - NumScannerThreadsStarted: 2 (2)
E              - PeakMemoryUsage: 49.05 MB (51436544)
E              - PerReadThreadRawHdfsThroughput: 1.72 GB/sec
E              - RemoteScanRanges: 0 (0)
E              - RowBatchQueueGetWaitTime: 0.000ns
E              - RowBatchQueuePutWaitTime: 0.000ns
E              - RowsRead: 556.03K (556032)
E              - RowsReturned: 500.74K (500736)
E              - RowsReturnedRate: 7.82 M/sec
E              - ScanRangesComplete: 0 (0)
E              - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E              - ScannerThreadsTotalWallClockTime: 0.000ns
E                - DelimiterParseTime: 243.004ms
E                - MaterializeTupleTime(*): 26.000ms
E                - ScannerThreadsSysTime: 0.000ns
E                - ScannerThreadsUserTime: 0.000ns
E              - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E              - TotalRawHdfsOpenFileTime(*): 0.000ns
E              - TotalRawHdfsReadTime(*): 59.001ms
E              - TotalReadThroughput: 80.00 MB/sec
E           CodeGen:(Total: 265.005ms, non-child: 265.005ms, % non-child: 100.00%)
E              - CodegenInvoluntaryContextSwitches: 221 (221)
E              - CodegenTotalWallClockTime: 265.005ms
E                - CodegenSysTime: 4.999ms
E                - CodegenUserTime: 142.979ms
E              - CodegenVoluntaryContextSwitches: 8 (8)
E              - CompileTime: 49.000ms
E              - IrGenerationTime: 3.000ms
E              - LoadTime: 0.000ns
E              - ModuleBitcodeSize: 2.26 MB (2373956)
E              - NumFunctions: 30 (30)
E              - NumInstructions: 562 (562)
E              - OptimizationTime: 170.003ms
E              - PeakMemoryUsage: 281.00 KB (287744)
E              - PrepareTime: 42.000ms
E         Instance 31479a87f057d480:fb4b93dc00000005 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22001):(Total: 1s160ms, non-child: 3.000ms, % non-child: 0.26%)
E           Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:2/256.00 MB 0:2/256.00 MB 
E           Fragment Instance Lifecycle Event Timeline: 1s163ms
E              - Prepare Finished: 2.000ms (2.000ms)
E              - Open Finished: 169.003ms (167.003ms)
E              - First Batch Produced: 177.003ms (8.000ms)
E              - First Batch Sent: 177.003ms (0.000ns)
E              - ExecInternal Finished: 1s163ms (986.019ms)
E           MemoryUsage(500.000ms): 303.22 KB, 49.07 MB, 49.07 MB
E           ThreadUsage(500.000ms): 1, 3, 3
E            - AverageThreadTokens: 2.33 
E            - BloomFilterBytes: 0
E            - PeakMemoryUsage: 49.09 MB (51471608)
E            - PeakReservation: 0
E            - PeakUsedReservation: 0
E            - PerHostPeakMemUsage: 97.21 MB (101935936)
E            - RowsProduced: 500.00K (500000)
E            - TotalNetworkReceiveTime: 0.000ns
E            - TotalNetworkSendTime: 853.016ms
E            - TotalStorageWaitTime: 5.000ms
E            - TotalThreadsInvoluntaryContextSwitches: 163 (163)
E            - TotalThreadsTotalWallClockTime: 1s160ms
E              - TotalThreadsSysTime: 36.994ms
E              - TotalThreadsUserTime: 181.973ms
E            - TotalThreadsVoluntaryContextSwitches: 1.01K (1013)
E           Buffer pool:
E              - AllocTime: 0.000ns
E              - CumulativeAllocationBytes: 0
E              - CumulativeAllocations: 0 (0)
E              - PeakReservation: 0
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 0
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - ReservationLimit: 0
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Fragment Instance Lifecycle Timings:
E              - ExecTime: 994.019ms
E                - ExecTreeExecTime: 98.001ms
E              - OpenTime: 166.003ms
E                - ExecTreeOpenTime: 0.000ns
E              - PrepareTime: 0.000ns
E                - ExecTreePrepareTime: 0.000ns
E           KrpcDataStreamSender (dst_id=7):(Total: 894.017ms, non-child: 41.000ms, % non-child: 4.59%)
E             BytesSent(500.000ms): 0, 1.66 MB, 2.27 MB
E              - EosSent: 0 (0)
E              - NetworkThroughput: 6.57 MB/sec
E              - PeakMemoryUsage: 14.22 KB (14560)
E              - RowsSent: 500.00K (500000)
E              - RpcFailure: 0 (0)
E              - RpcRetry: 0 (0)
E              - SerializeBatchTime: 23.000ms
E              - TotalBytesSent: 2.28 MB (2394167)
E              - UncompressedRowBatchSize: 8.11 MB (8500000)
E           UNION_NODE (id=1):(Total: 97.001ms, non-child: 0.000ns, % non-child: 0.00%)
E             ExecOption: Codegen Enabled
E              - PeakMemoryUsage: 0
E              - RowsReturned: 500.00K (500000)
E              - RowsReturnedRate: 5.15 M/sec
E             HDFS_SCAN_NODE (id=3):
E               Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/128.00 MB 0:1/128.00 MB 
E               ExecOption: TEXT Codegen Enabled
E               BytesRead(500.000ms): 0, 0, 0
E                - BytesRead: 0
E                - CollectionItemsRead: 0 (0)
E                - PeakMemoryUsage: 0
E                - RowBatchQueueGetWaitTime: 0.000ns
E                - RowBatchQueuePutWaitTime: 0.000ns
E                - RowsRead: 0 (0)
E                - RowsReturned: 0 (0)
E                - RowsReturnedRate: 0
E                - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E                - ScannerThreadsTotalWallClockTime: 0.000ns
E                  - MaterializeTupleTime(*): 0.000ns
E                  - ScannerThreadsSysTime: 0.000ns
E                  - ScannerThreadsUserTime: 0.000ns
E                - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E                - TotalReadThroughput: 0.00 /sec
E           HDFS_SCAN_NODE (id=2):(Total: 97.001ms, non-child: 97.001ms, % non-child: 100.00%)
E             Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/128.00 MB 0:1/128.00 MB 
E             ExecOption: TEXT Codegen Enabled
E             BytesRead(500.000ms): 0, 80.00 MB, 96.00 MB
E              - AverageHdfsReadThreadConcurrency: 0.00 
E              - AverageScannerThreadConcurrency: 2.00 
E              - BytesRead: 96.00 MB (100663296)
E              - BytesReadDataNodeCache: 0
E              - BytesReadLocal: 0
E              - BytesReadRemoteUnexpected: 0
E              - BytesReadShortCircuit: 0
E              - CachedFileHandlesHitCount: 0 (0)
E              - CachedFileHandlesMissCount: 0 (0)
E              - CollectionItemsRead: 0 (0)
E              - DecompressionTime: 0.000ns
E              - MaxCompressedTextFileLength: 0
E              - NumDisksAccessed: 0 (0)
E              - NumScannerThreadsStarted: 2 (2)
E              - PeakMemoryUsage: 49.05 MB (51437568)
E              - PerReadThreadRawHdfsThroughput: 1.87 GB/sec
E              - RemoteScanRanges: 0 (0)
E              - RowBatchQueueGetWaitTime: 0.000ns
E              - RowBatchQueuePutWaitTime: 0.000ns
E              - RowsRead: 519.17K (519168)
E              - RowsReturned: 500.74K (500736)
E              - RowsReturnedRate: 5.16 M/sec
E              - ScanRangesComplete: 0 (0)
E              - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E              - ScannerThreadsTotalWallClockTime: 0.000ns
E                - DelimiterParseTime: 223.004ms
E                - MaterializeTupleTime(*): 151.002ms
E                - ScannerThreadsSysTime: 0.000ns
E                - ScannerThreadsUserTime: 0.000ns
E              - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E              - TotalRawHdfsOpenFileTime(*): 0.000ns
E              - TotalRawHdfsReadTime(*): 50.000ms
E              - TotalReadThroughput: 64.00 MB/sec
E           CodeGen:(Total: 166.003ms, non-child: 166.003ms, % non-child: 100.00%)
E              - CodegenInvoluntaryContextSwitches: 157 (157)
E              - CodegenTotalWallClockTime: 166.003ms
E                - CodegenSysTime: 7.998ms
E                - CodegenUserTime: 132.980ms
E              - CodegenVoluntaryContextSwitches: 6 (6)
E              - CompileTime: 46.000ms
E              - IrGenerationTime: 5.000ms
E              - LoadTime: 0.000ns
E              - ModuleBitcodeSize: 2.26 MB (2373956)
E              - NumFunctions: 30 (30)
E              - NumInstructions: 562 (562)
E              - OptimizationTime: 80.001ms
E              - PeakMemoryUsage: 281.00 KB (287744)
E              - PrepareTime: 35.000ms
E         Instance 31479a87f057d480:fb4b93dc00000006 (host=ec2-m2-4xlarge-centos-6-4-0716.vpc.cloudera.com:22002):(Total: 1s146ms, non-child: 1.000ms, % non-child: 0.09%)
E           Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:2/157.88 MB 0:2/256.00 MB 
E           Fragment Instance Lifecycle Event Timeline: 1s148ms
E              - Prepare Finished: 2.000ms (2.000ms)
E              - Open Finished: 179.003ms (177.003ms)
E              - First Batch Produced: 193.003ms (14.000ms)
E              - First Batch Sent: 193.003ms (0.000ns)
E              - ExecInternal Finished: 1s148ms (955.018ms)
E           MemoryUsage(500.000ms): 49.05 MB, 49.07 MB
E           ThreadUsage(500.000ms): 3, 3
E            - AverageThreadTokens: 3.00 
E            - BloomFilterBytes: 0
E            - PeakMemoryUsage: 49.09 MB (51479800)
E            - PeakReservation: 0
E            - PeakUsedReservation: 0
E            - PerHostPeakMemUsage: 97.23 MB (101953344)
E            - RowsProduced: 500.00K (500000)
E            - TotalNetworkReceiveTime: 0.000ns
E            - TotalNetworkSendTime: 904.017ms
E            - TotalStorageWaitTime: 5.000ms
E            - TotalThreadsInvoluntaryContextSwitches: 197 (197)
E            - TotalThreadsTotalWallClockTime: 1s146ms
E              - TotalThreadsSysTime: 33.994ms
E              - TotalThreadsUserTime: 178.972ms
E            - TotalThreadsVoluntaryContextSwitches: 889 (889)
E           Buffer pool:
E              - AllocTime: 0.000ns
E              - CumulativeAllocationBytes: 0
E              - CumulativeAllocations: 0 (0)
E              - PeakReservation: 0
E              - PeakUnpinnedBytes: 0
E              - PeakUsedReservation: 0
E              - ReadIoBytes: 0
E              - ReadIoOps: 0 (0)
E              - ReadIoWaitTime: 0.000ns
E              - ReservationLimit: 0
E              - WriteIoBytes: 0
E              - WriteIoOps: 0 (0)
E              - WriteIoWaitTime: 0.000ns
E           Fragment Instance Lifecycle Timings:
E              - ExecTime: 969.018ms
E                - ExecTreeExecTime: 28.000ms
E              - OpenTime: 177.003ms
E                - ExecTreeOpenTime: 0.000ns
E              - PrepareTime: 0.000ns
E                - ExecTreePrepareTime: 0.000ns
E           KrpcDataStreamSender (dst_id=7):(Total: 940.018ms, non-child: 36.000ms, % non-child: 3.83%)
E             BytesSent(500.000ms): 811.76 KB, 2.33 MB
E              - EosSent: 0 (0)
E              - NetworkThroughput: 6.42 MB/sec
E              - PeakMemoryUsage: 14.22 KB (14560)
E              - RowsSent: 500.00K (500000)
E              - RpcFailure: 0 (0)
E              - RpcRetry: 0 (0)
E              - SerializeBatchTime: 29.000ms
E              - TotalBytesSent: 2.34 MB (2453058)
E              - UncompressedRowBatchSize: 8.11 MB (8500000)
E           UNION_NODE (id=1):(Total: 28.000ms, non-child: 0.000ns, % non-child: 0.00%)
E             ExecOption: Codegen Enabled
E              - PeakMemoryUsage: 0
E              - RowsReturned: 500.00K (500000)
E              - RowsReturnedRate: 17.86 M/sec
E             HDFS_SCAN_NODE (id=3):
E               Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/78.94 MB 0:1/128.00 MB 
E               ExecOption: TEXT Codegen Enabled
E               BytesRead(500.000ms): 0, 0
E                - BytesRead: 0
E                - CollectionItemsRead: 0 (0)
E                - PeakMemoryUsage: 0
E                - RowBatchQueueGetWaitTime: 0.000ns
E                - RowBatchQueuePutWaitTime: 0.000ns
E                - RowsRead: 0 (0)
E                - RowsReturned: 0 (0)
E                - RowsReturnedRate: 0
E                - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E                - ScannerThreadsTotalWallClockTime: 0.000ns
E                  - MaterializeTupleTime(*): 0.000ns
E                  - ScannerThreadsSysTime: 0.000ns
E                  - ScannerThreadsUserTime: 0.000ns
E                - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E                - TotalReadThroughput: 0.00 /sec
E           HDFS_SCAN_NODE (id=2):(Total: 28.000ms, non-child: 28.000ms, % non-child: 100.00%)
E             Hdfs split stats (<volume id>:<# splits>/<split lengths>): 2:1/78.94 MB 0:1/128.00 MB 
E             ExecOption: TEXT Codegen Enabled
E             BytesRead(500.000ms): 64.00 MB, 96.00 MB
E              - AverageHdfsReadThreadConcurrency: 0.00 
E              - AverageScannerThreadConcurrency: 2.00 
E              - BytesRead: 96.00 MB (100663296)
E              - BytesReadDataNodeCache: 0
E              - BytesReadLocal: 0
E              - BytesReadRemoteUnexpected: 0
E              - BytesReadShortCircuit: 0
E              - CachedFileHandlesHitCount: 0 (0)
E              - CachedFileHandlesMissCount: 0 (0)
E              - CollectionItemsRead: 0 (0)
E              - DecompressionTime: 0.000ns
E              - MaxCompressedTextFileLength: 0
E              - NumDisksAccessed: 0 (0)
E              - NumScannerThreadsStarted: 2 (2)
E              - PeakMemoryUsage: 49.07 MB (51454976)
E              - PerReadThreadRawHdfsThroughput: 1.40 GB/sec
E              - RemoteScanRanges: 0 (0)
E              - RowBatchQueueGetWaitTime: 0.000ns
E              - RowBatchQueuePutWaitTime: 0.000ns
E              - RowsRead: 531.46K (531456)
E              - RowsReturned: 500.74K (500736)
E              - RowsReturnedRate: 17.88 M/sec
E              - ScanRangesComplete: 0 (0)
E              - ScannerThreadsInvoluntaryContextSwitches: 0 (0)
E              - ScannerThreadsTotalWallClockTime: 0.000ns
E                - DelimiterParseTime: 265.005ms
E                - MaterializeTupleTime(*): 18.000ms
E                - ScannerThreadsSysTime: 0.000ns
E                - ScannerThreadsUserTime: 0.000ns
E              - ScannerThreadsVoluntaryContextSwitches: 0 (0)
E              - TotalRawHdfsOpenFileTime(*): 1.000ms
E              - TotalRawHdfsReadTime(*): 67.001ms
E              - TotalReadThroughput: 96.00 MB/sec
E           CodeGen:(Total: 177.003ms, non-child: 177.003ms, % non-child: 100.00%)
E              - CodegenInvoluntaryContextSwitches: 186 (186)
E              - CodegenTotalWallClockTime: 177.003ms
E                - CodegenSysTime: 8.998ms
E                - CodegenUserTime: 130.979ms
E              - CodegenVoluntaryContextSwitches: 9 (9)
E              - CompileTime: 58.001ms
E              - IrGenerationTime: 3.000ms
E              - LoadTime: 0.000ns
E              - ModuleBitcodeSize: 2.26 MB (2373956)
E              - NumFunctions: 30 (30)
E              - NumInstructions: 562 (562)
E              - OptimizationTime: 91.001ms
E              - PeakMemoryUsage: 281.00 KB (287744)
E              - PrepareTime: 24.000ms{noformat}



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