Uploaded image for project: 'Hive'
  1. Hive
  2. HIVE-13027

Configuration changes to improve logging performance

    Details

    • Type: Improvement
    • Status: Closed
    • Priority: Major
    • Resolution: Fixed
    • Affects Version/s: 2.1.0
    • Fix Version/s: 2.1.0
    • Component/s: Logging
    • Labels:
      None

      Description

      For LLAP and HS2, some configuration changes can be made to improve logging performance
      1) LOG4j2's async logger claims to have 6-68 times better performance than synchronous logger. https://logging.apache.org/log4j/2.x/manual/async.html
      2) Replace File appenders with RandomAccessFileAppender that claims to be 20-200% more performant.
      https://logging.apache.org/log4j/2.x/manual/appenders.html#RandomAccessFileAppender

      Also make async logging configurable.

      1. HIVE-13027.1.patch
        3 kB
        Prasanth Jayachandran
      2. HIVE-13027.2.patch
        19 kB
        Prasanth Jayachandran
      3. HIVE-13027.3.patch
        22 kB
        Prasanth Jayachandran
      4. HIVE-13027.3.patch
        22 kB
        Prasanth Jayachandran
      5. HIVE-13027.4.patch
        19 kB
        Prasanth Jayachandran

        Issue Links

          Activity

          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Gopal V For using Async loggers, we need disruptor.jar in classpath. Currently disruptor.jar is pulled transitively by hbase-server. Do we need to package disruptor in hive-exec?

          Show
          prasanth_j Prasanth Jayachandran added a comment - Gopal V For using Async loggers, we need disruptor.jar in classpath. Currently disruptor.jar is pulled transitively by hbase-server. Do we need to package disruptor in hive-exec?
          Hide
          hiveqa Hive QA added a comment -

          Here are the results of testing the latest attachment:
          https://issues.apache.org/jira/secure/attachment/12786982/HIVE-13027.1.patch

          ERROR: -1 due to no test(s) being added or modified.

          ERROR: -1 due to 3 failed/errored test(s), 9766 tests executed
          Failed tests:

          TestCompareCliDriver - did not produce a TEST-*.xml file
          org.apache.hadoop.hive.cli.TestNegativeCliDriver.testNegativeCliDriver_authorization_uri_import
          org.apache.hive.jdbc.TestSSL.testSSLVersion
          

          Test results: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/6928/testReport
          Console output: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/6928/console
          Test logs: http://ec2-174-129-184-35.compute-1.amazonaws.com/logs/PreCommit-HIVE-TRUNK-Build-6928/

          Messages:

          Executing org.apache.hive.ptest.execution.TestCheckPhase
          Executing org.apache.hive.ptest.execution.PrepPhase
          Executing org.apache.hive.ptest.execution.ExecutionPhase
          Executing org.apache.hive.ptest.execution.ReportingPhase
          Tests exited with: TestsFailedException: 3 tests failed
          

          This message is automatically generated.

          ATTACHMENT ID: 12786982 - PreCommit-HIVE-TRUNK-Build

          Show
          hiveqa Hive QA added a comment - Here are the results of testing the latest attachment: https://issues.apache.org/jira/secure/attachment/12786982/HIVE-13027.1.patch ERROR: -1 due to no test(s) being added or modified. ERROR: -1 due to 3 failed/errored test(s), 9766 tests executed Failed tests: TestCompareCliDriver - did not produce a TEST-*.xml file org.apache.hadoop.hive.cli.TestNegativeCliDriver.testNegativeCliDriver_authorization_uri_import org.apache.hive.jdbc.TestSSL.testSSLVersion Test results: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/6928/testReport Console output: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/6928/console Test logs: http://ec2-174-129-184-35.compute-1.amazonaws.com/logs/PreCommit-HIVE-TRUNK-Build-6928/ Messages: Executing org.apache.hive.ptest.execution.TestCheckPhase Executing org.apache.hive.ptest.execution.PrepPhase Executing org.apache.hive.ptest.execution.ExecutionPhase Executing org.apache.hive.ptest.execution.ReportingPhase Tests exited with: TestsFailedException: 3 tests failed This message is automatically generated. ATTACHMENT ID: 12786982 - PreCommit-HIVE-TRUNK-Build
          Hide
          gopalv Gopal V added a comment -

          Prasanth Jayachandran: yes, some jars to be bundled - HIVE-13015 (see .3 patch).

          Confirmed a run of TPC-DS Q55 using 1 node with INFO logging ran in ~0.6 seconds, but my INFO logs are going to console.

          Trying to figure out how the llap log4j is loaded up by the Configurator (is it still reading hive-exec-log4j2.properties?).

          Show
          gopalv Gopal V added a comment - Prasanth Jayachandran : yes, some jars to be bundled - HIVE-13015 (see .3 patch). Confirmed a run of TPC-DS Q55 using 1 node with INFO logging ran in ~0.6 seconds, but my INFO logs are going to console. Trying to figure out how the llap log4j is loaded up by the Configurator (is it still reading hive-exec-log4j2.properties?).
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          The automatic log4j2 configurator will look for file specified via -Dlog4j.configurationFile.. if this file could not be located in classpath then it will default to ERROR,console.

          Programmatic initialization of logging happens in CliDriver and when running metastore as a service. Programmatic initialization looks for hive-log4j2.properties. mr/ExecDriver which initializes the logging for child JVM uses hive-exec-log4j2.properties. I suspect initialization using hive-exec-log4j2.properties as it is mr only code path.

          Show
          prasanth_j Prasanth Jayachandran added a comment - The automatic log4j2 configurator will look for file specified via -Dlog4j.configurationFile.. if this file could not be located in classpath then it will default to ERROR,console. Programmatic initialization of logging happens in CliDriver and when running metastore as a service. Programmatic initialization looks for hive-log4j2.properties. mr/ExecDriver which initializes the logging for child JVM uses hive-exec-log4j2.properties. I suspect initialization using hive-exec-log4j2.properties as it is mr only code path.
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          I tried another query with TPCDS 1TB scale. The runtime for queries at WARN level and INFO are 18.1s vs 18.5s respectively. One thing I am noticing is that the presence of disruptor jar in the classpath itself triggers Async logging without -DLog4jContextSelector system property. I don't know why yet.

          Show
          prasanth_j Prasanth Jayachandran added a comment - I tried another query with TPCDS 1TB scale. The runtime for queries at WARN level and INFO are 18.1s vs 18.5s respectively. One thing I am noticing is that the presence of disruptor jar in the classpath itself triggers Async logging without -DLog4jContextSelector system property. I don't know why yet.
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Attaching RB link. Gopal V Can you plz review the patch?

          Show
          prasanth_j Prasanth Jayachandran added a comment - Attaching RB link. Gopal V Can you plz review the patch?
          Hide
          hiveqa Hive QA added a comment -

          Here are the results of testing the latest attachment:
          https://issues.apache.org/jira/secure/attachment/12794710/HIVE-13027.2.patch

          SUCCESS: +1 due to 1 test(s) being added or modified.

          ERROR: -1 due to 25 failed/errored test(s), 9857 tests executed
          Failed tests:

          TestSparkCliDriver-groupby3_map.q-sample2.q-auto_join14.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-groupby_map_ppr_multi_distinct.q-table_access_keys_stats.q-groupby4_noskew.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-join_rc.q-insert1.q-vectorized_rcfile_columnar.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-ppd_join4.q-join9.q-ppd_join3.q-and-12-more - did not produce a TEST-*.xml file
          org.apache.hadoop.hive.cli.TestCliDriverMethods.testProcessSelectDatabase
          org.apache.hadoop.hive.cli.TestCliDriverMethods.testQuit
          org.apache.hadoop.hive.cli.TestCliDriverMethods.testRun
          org.apache.hadoop.hive.cli.TestCliDriverMethods.testThatCliDriverPrintsHeaderForCommandsWithSchema
          org.apache.hadoop.hive.cli.TestCliDriverMethods.testThatCliDriverPrintsNoHeaderForCommandsWithNoSchema
          org.apache.hadoop.hive.cli.TestCliDriverMethods.testprocessInitFiles
          org.apache.hadoop.hive.cli.TestContribNegativeCliDriver.testNegativeCliDriver_case_with_row_sequence
          org.apache.hadoop.hive.cli.TestMiniLlapCliDriver.testCliDriver_llap_udf
          org.apache.hadoop.hive.metastore.txn.TestCompactionTxnHandler.testRevokeTimedOutWorkers
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapPlan1
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapPlan2
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan1
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan2
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan3
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan4
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan5
          org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan6
          org.apache.hadoop.hive.ql.lockmgr.TestDbTxnManager.concurrencyFalse
          org.apache.hadoop.hive.ql.lockmgr.TestDbTxnManager.testLockTimeout
          org.apache.hadoop.hive.ql.lockmgr.TestDbTxnManager.testUpdate
          org.apache.hive.hcatalog.cli.TestPermsGrp.testCustomPerms
          

          Test results: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7355/testReport
          Console output: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7355/console
          Test logs: http://ec2-174-129-184-35.compute-1.amazonaws.com/logs/PreCommit-HIVE-TRUNK-Build-7355/

          Messages:

          Executing org.apache.hive.ptest.execution.TestCheckPhase
          Executing org.apache.hive.ptest.execution.PrepPhase
          Executing org.apache.hive.ptest.execution.ExecutionPhase
          Executing org.apache.hive.ptest.execution.ReportingPhase
          Tests exited with: TestsFailedException: 25 tests failed
          

          This message is automatically generated.

          ATTACHMENT ID: 12794710 - PreCommit-HIVE-TRUNK-Build

          Show
          hiveqa Hive QA added a comment - Here are the results of testing the latest attachment: https://issues.apache.org/jira/secure/attachment/12794710/HIVE-13027.2.patch SUCCESS: +1 due to 1 test(s) being added or modified. ERROR: -1 due to 25 failed/errored test(s), 9857 tests executed Failed tests: TestSparkCliDriver-groupby3_map.q-sample2.q-auto_join14.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-groupby_map_ppr_multi_distinct.q-table_access_keys_stats.q-groupby4_noskew.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-join_rc.q-insert1.q-vectorized_rcfile_columnar.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-ppd_join4.q-join9.q-ppd_join3.q-and-12-more - did not produce a TEST-*.xml file org.apache.hadoop.hive.cli.TestCliDriverMethods.testProcessSelectDatabase org.apache.hadoop.hive.cli.TestCliDriverMethods.testQuit org.apache.hadoop.hive.cli.TestCliDriverMethods.testRun org.apache.hadoop.hive.cli.TestCliDriverMethods.testThatCliDriverPrintsHeaderForCommandsWithSchema org.apache.hadoop.hive.cli.TestCliDriverMethods.testThatCliDriverPrintsNoHeaderForCommandsWithNoSchema org.apache.hadoop.hive.cli.TestCliDriverMethods.testprocessInitFiles org.apache.hadoop.hive.cli.TestContribNegativeCliDriver.testNegativeCliDriver_case_with_row_sequence org.apache.hadoop.hive.cli.TestMiniLlapCliDriver.testCliDriver_llap_udf org.apache.hadoop.hive.metastore.txn.TestCompactionTxnHandler.testRevokeTimedOutWorkers org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapPlan1 org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapPlan2 org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan1 org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan2 org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan3 org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan4 org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan5 org.apache.hadoop.hive.ql.exec.TestExecDriver.testMapRedPlan6 org.apache.hadoop.hive.ql.lockmgr.TestDbTxnManager.concurrencyFalse org.apache.hadoop.hive.ql.lockmgr.TestDbTxnManager.testLockTimeout org.apache.hadoop.hive.ql.lockmgr.TestDbTxnManager.testUpdate org.apache.hive.hcatalog.cli.TestPermsGrp.testCustomPerms Test results: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7355/testReport Console output: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7355/console Test logs: http://ec2-174-129-184-35.compute-1.amazonaws.com/logs/PreCommit-HIVE-TRUNK-Build-7355/ Messages: Executing org.apache.hive.ptest.execution.TestCheckPhase Executing org.apache.hive.ptest.execution.PrepPhase Executing org.apache.hive.ptest.execution.ExecutionPhase Executing org.apache.hive.ptest.execution.ReportingPhase Tests exited with: TestsFailedException: 25 tests failed This message is automatically generated. ATTACHMENT ID: 12794710 - PreCommit-HIVE-TRUNK-Build
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Gopal V Add appending to llap daemon logs. Also for test failures included disruptor jars in pom.xml. Should we pack the disruptor jar in hive-exec instead or can we assume that it will be pulled in transitively and be available in cluster?

          Show
          prasanth_j Prasanth Jayachandran added a comment - Gopal V Add appending to llap daemon logs. Also for test failures included disruptor jars in pom.xml. Should we pack the disruptor jar in hive-exec instead or can we assume that it will be pulled in transitively and be available in cluster?
          Hide
          hiveqa Hive QA added a comment -

          Here are the results of testing the latest attachment:
          https://issues.apache.org/jira/secure/attachment/12796003/HIVE-13027.3.patch

          SUCCESS: +1 due to 1 test(s) being added or modified.

          ERROR: -1 due to 8 failed/errored test(s), 9905 tests executed
          Failed tests:

          TestSparkCliDriver-groupby_map_ppr_multi_distinct.q-groupby10.q-timestamp_comparison.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-parallel_join0.q-union_remove_9.q-smb_mapjoin_21.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-ppd_join4.q-join9.q-ppd_join3.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-timestamp_lazy.q-date_udf.q-auto_join4.q-and-12-more - did not produce a TEST-*.xml file
          org.apache.hadoop.hive.cli.TestContribNegativeCliDriver.testNegativeCliDriver_case_with_row_sequence
          org.apache.hadoop.hive.cli.TestMiniSparkOnYarnCliDriver.testCliDriver_index_bitmap3
          org.apache.hadoop.hive.metastore.TestHiveMetaStorePartitionSpecs.testGetPartitionSpecs_WithAndWithoutPartitionGrouping
          org.apache.hive.hcatalog.hbase.TestPigHBaseStorageHandler.org.apache.hive.hcatalog.hbase.TestPigHBaseStorageHandler
          

          Test results: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7429/testReport
          Console output: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7429/console
          Test logs: http://ec2-174-129-184-35.compute-1.amazonaws.com/logs/PreCommit-HIVE-TRUNK-Build-7429/

          Messages:

          Executing org.apache.hive.ptest.execution.TestCheckPhase
          Executing org.apache.hive.ptest.execution.PrepPhase
          Executing org.apache.hive.ptest.execution.ExecutionPhase
          Executing org.apache.hive.ptest.execution.ReportingPhase
          Tests exited with: TestsFailedException: 8 tests failed
          

          This message is automatically generated.

          ATTACHMENT ID: 12796003 - PreCommit-HIVE-TRUNK-Build

          Show
          hiveqa Hive QA added a comment - Here are the results of testing the latest attachment: https://issues.apache.org/jira/secure/attachment/12796003/HIVE-13027.3.patch SUCCESS: +1 due to 1 test(s) being added or modified. ERROR: -1 due to 8 failed/errored test(s), 9905 tests executed Failed tests: TestSparkCliDriver-groupby_map_ppr_multi_distinct.q-groupby10.q-timestamp_comparison.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-parallel_join0.q-union_remove_9.q-smb_mapjoin_21.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-ppd_join4.q-join9.q-ppd_join3.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-timestamp_lazy.q-date_udf.q-auto_join4.q-and-12-more - did not produce a TEST-*.xml file org.apache.hadoop.hive.cli.TestContribNegativeCliDriver.testNegativeCliDriver_case_with_row_sequence org.apache.hadoop.hive.cli.TestMiniSparkOnYarnCliDriver.testCliDriver_index_bitmap3 org.apache.hadoop.hive.metastore.TestHiveMetaStorePartitionSpecs.testGetPartitionSpecs_WithAndWithoutPartitionGrouping org.apache.hive.hcatalog.hbase.TestPigHBaseStorageHandler.org.apache.hive.hcatalog.hbase.TestPigHBaseStorageHandler Test results: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7429/testReport Console output: http://ec2-174-129-184-35.compute-1.amazonaws.com/jenkins/job/PreCommit-HIVE-TRUNK-Build/7429/console Test logs: http://ec2-174-129-184-35.compute-1.amazonaws.com/logs/PreCommit-HIVE-TRUNK-Build-7429/ Messages: Executing org.apache.hive.ptest.execution.TestCheckPhase Executing org.apache.hive.ptest.execution.PrepPhase Executing org.apache.hive.ptest.execution.ExecutionPhase Executing org.apache.hive.ptest.execution.ReportingPhase Tests exited with: TestsFailedException: 8 tests failed This message is automatically generated. ATTACHMENT ID: 12796003 - PreCommit-HIVE-TRUNK-Build
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Sergey Shelukhin Can you please review this? This is pending for a long time.

          Show
          prasanth_j Prasanth Jayachandran added a comment - Sergey Shelukhin Can you please review this? This is pending for a long time.
          Hide
          sershe Sergey Shelukhin added a comment -

          +1 assuming it actually works and logs to correct files

          Show
          sershe Sergey Shelukhin added a comment - +1 assuming it actually works and logs to correct files
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Rebased patch for precommit test run

          Show
          prasanth_j Prasanth Jayachandran added a comment - Rebased patch for precommit test run
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Some properties like append, immediateFlush and includeLocation are being set to the defaults. It is redundant. Hence removed in this patch.

          Show
          prasanth_j Prasanth Jayachandran added a comment - Some properties like append, immediateFlush and includeLocation are being set to the defaults. It is redundant. Hence removed in this patch.
          Hide
          hiveqa Hive QA added a comment -

          Here are the results of testing the latest attachment:
          https://issues.apache.org/jira/secure/attachment/12802101/HIVE-13027.4.patch

          SUCCESS: +1 due to 1 test(s) being added or modified.

          ERROR: -1 due to 16 failed/errored test(s), 9918 tests executed
          Failed tests:

          TestCliDriver-gen_udf_example_add10.q-ppd_join4.q-union27.q-and-12-more - did not produce a TEST-*.xml file
          TestCliDriver-partition_timestamp.q-ppd_random.q-vector_outer_join5.q-and-12-more - did not produce a TEST-*.xml file
          TestCliDriver-ptf_general_queries.q-unionDistinct_1.q-groupby1_noskew.q-and-12-more - did not produce a TEST-*.xml file
          TestHWISessionManager - did not produce a TEST-*.xml file
          TestMiniLlapCliDriver - did not produce a TEST-*.xml file
          TestMiniTezCliDriver-auto_sortmerge_join_7.q-tez_union_group_by.q-orc_merge9.q-and-12-more - did not produce a TEST-*.xml file
          TestMiniTezCliDriver-enforce_order.q-vector_partition_diff_num_cols.q-unionDistinct_1.q-and-12-more - did not produce a TEST-*.xml file
          TestMiniTezCliDriver-insert_values_non_partitioned.q-join1.q-schema_evol_orc_nonvec_mapwork_part.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-bucketsortoptimize_insert_7.q-smb_mapjoin_15.q-mapreduce1.q-and-12-more - did not produce a TEST-*.xml file
          TestSparkCliDriver-skewjoinopt3.q-union27.q-multigroupby_singlemr.q-and-12-more - did not produce a TEST-*.xml file
          org.apache.hadoop.hive.cli.TestContribNegativeCliDriver.testNegativeCliDriver_case_with_row_sequence
          org.apache.hadoop.hive.cli.TestMiniSparkOnYarnCliDriver.testCliDriver_index_bitmap3
          org.apache.hadoop.hive.llap.tez.TestConverters.testFragmentSpecToTaskSpec
          org.apache.hadoop.hive.llap.tezplugins.TestLlapTaskCommunicator.testFinishableStateUpdateFailure
          org.apache.hadoop.hive.ql.security.authorization.plugin.TestHiveOperationType.checkHiveOperationTypeMatch
          org.apache.hive.service.cli.session.TestHiveSessionImpl.testLeakOperationHandle
          

          Test results: http://ec2-54-177-240-2.us-west-1.compute.amazonaws.com/job/PreCommit-HIVE-MASTER-Build/192/testReport
          Console output: http://ec2-54-177-240-2.us-west-1.compute.amazonaws.com/job/PreCommit-HIVE-MASTER-Build/192/console
          Test logs: http://ec2-50-18-27-0.us-west-1.compute.amazonaws.com/logs/PreCommit-HIVE-MASTER-Build-192/

          Messages:

          Executing org.apache.hive.ptest.execution.TestCheckPhase
          Executing org.apache.hive.ptest.execution.PrepPhase
          Executing org.apache.hive.ptest.execution.ExecutionPhase
          Executing org.apache.hive.ptest.execution.ReportingPhase
          Tests exited with: TestsFailedException: 16 tests failed
          

          This message is automatically generated.

          ATTACHMENT ID: 12802101 - PreCommit-HIVE-MASTER-Build

          Show
          hiveqa Hive QA added a comment - Here are the results of testing the latest attachment: https://issues.apache.org/jira/secure/attachment/12802101/HIVE-13027.4.patch SUCCESS: +1 due to 1 test(s) being added or modified. ERROR: -1 due to 16 failed/errored test(s), 9918 tests executed Failed tests: TestCliDriver-gen_udf_example_add10.q-ppd_join4.q-union27.q-and-12-more - did not produce a TEST-*.xml file TestCliDriver-partition_timestamp.q-ppd_random.q-vector_outer_join5.q-and-12-more - did not produce a TEST-*.xml file TestCliDriver-ptf_general_queries.q-unionDistinct_1.q-groupby1_noskew.q-and-12-more - did not produce a TEST-*.xml file TestHWISessionManager - did not produce a TEST-*.xml file TestMiniLlapCliDriver - did not produce a TEST-*.xml file TestMiniTezCliDriver-auto_sortmerge_join_7.q-tez_union_group_by.q-orc_merge9.q-and-12-more - did not produce a TEST-*.xml file TestMiniTezCliDriver-enforce_order.q-vector_partition_diff_num_cols.q-unionDistinct_1.q-and-12-more - did not produce a TEST-*.xml file TestMiniTezCliDriver-insert_values_non_partitioned.q-join1.q-schema_evol_orc_nonvec_mapwork_part.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-bucketsortoptimize_insert_7.q-smb_mapjoin_15.q-mapreduce1.q-and-12-more - did not produce a TEST-*.xml file TestSparkCliDriver-skewjoinopt3.q-union27.q-multigroupby_singlemr.q-and-12-more - did not produce a TEST-*.xml file org.apache.hadoop.hive.cli.TestContribNegativeCliDriver.testNegativeCliDriver_case_with_row_sequence org.apache.hadoop.hive.cli.TestMiniSparkOnYarnCliDriver.testCliDriver_index_bitmap3 org.apache.hadoop.hive.llap.tez.TestConverters.testFragmentSpecToTaskSpec org.apache.hadoop.hive.llap.tezplugins.TestLlapTaskCommunicator.testFinishableStateUpdateFailure org.apache.hadoop.hive.ql.security.authorization.plugin.TestHiveOperationType.checkHiveOperationTypeMatch org.apache.hive.service.cli.session.TestHiveSessionImpl.testLeakOperationHandle Test results: http://ec2-54-177-240-2.us-west-1.compute.amazonaws.com/job/PreCommit-HIVE-MASTER-Build/192/testReport Console output: http://ec2-54-177-240-2.us-west-1.compute.amazonaws.com/job/PreCommit-HIVE-MASTER-Build/192/console Test logs: http://ec2-50-18-27-0.us-west-1.compute.amazonaws.com/logs/PreCommit-HIVE-MASTER-Build-192/ Messages: Executing org.apache.hive.ptest.execution.TestCheckPhase Executing org.apache.hive.ptest.execution.PrepPhase Executing org.apache.hive.ptest.execution.ExecutionPhase Executing org.apache.hive.ptest.execution.ReportingPhase Tests exited with: TestsFailedException: 16 tests failed This message is automatically generated. ATTACHMENT ID: 12802101 - PreCommit-HIVE-MASTER-Build
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Test failures are not related to this patch. These 6 failures seems to be happening in previous run as well. Committed patch to master.

          Show
          prasanth_j Prasanth Jayachandran added a comment - Test failures are not related to this patch. These 6 failures seems to be happening in previous run as well. Committed patch to master.
          Hide
          leftylev Lefty Leverenz added a comment -

          Prasanth Jayachandran, does this need to be documented in the wiki?

          I ask because of your comment about async logging being added in 2.1 on the thread "hive 2.1.0 beeline cannot show verbose log" on user@hive.

          Show
          leftylev Lefty Leverenz added a comment - Prasanth Jayachandran , does this need to be documented in the wiki? I ask because of your comment about async logging being added in 2.1 on the thread "hive 2.1.0 beeline cannot show verbose log" on user@hive. user@hive – Re: hive 2.1.0 beeline cannot show verbose log Getting Started – Hive Logging
          Hide
          prasanth_j Prasanth Jayachandran added a comment -

          Yes. I updated the Hive Logging section with some description about asynchronous logging. Thanks Lefty Leverenz for catching this!

          Show
          prasanth_j Prasanth Jayachandran added a comment - Yes. I updated the Hive Logging section with some description about asynchronous logging. Thanks Lefty Leverenz for catching this!
          Hide
          leftylev Lefty Leverenz added a comment -

          Thanks Prasanth. I added hive.async.log.enabled to the Configuration Properties doc:

          Show
          leftylev Lefty Leverenz added a comment - Thanks Prasanth. I added hive.async.log.enabled to the Configuration Properties doc: Configuration Properties – hive.async.log.enabled

            People

            • Assignee:
              prasanth_j Prasanth Jayachandran
              Reporter:
              prasanth_j Prasanth Jayachandran
            • Votes:
              0 Vote for this issue
              Watchers:
              6 Start watching this issue

              Dates

              • Created:
                Updated:
                Resolved:

                Development