deniskuzZ commented on code in PR #5845:
URL: https://github.com/apache/hive/pull/5845#discussion_r2144560966


##########
ql/src/java/org/apache/hadoop/hive/ql/exec/vector/mapjoin/fast/VectorMapJoinFastHashTableLoader.java:
##########
@@ -307,9 +320,19 @@ public void load(MapJoinTableContainer[] mapJoinTables,
         LOG.info("Finished loading the queue for input: {} waiting {} minutes 
for TPool shutdown", inputName, 2);
         addQueueDoneSentinel();
         loadExecService.shutdown();
-
-        if (!loadExecService.awaitTermination(2, TimeUnit.MINUTES)) {
+        try {
+          CompletableFuture.allOf(loaderTasks.toArray(new 
CompletableFuture[0]))
+                  .get(2, TimeUnit.MINUTES);
+        } catch (TimeoutException e) {
           throw new HiveException("Failed to complete the hash table loader. 
Loading timed out.");
+        } catch (ExecutionException e) {
+          throw new HiveException("One of the loader threads failed", 
e.getCause());
+        } catch (InterruptedException e) {

Review Comment:
   please check the patch
   ````
   Subject: [PATCH] cleanup
   ---
   Index: 
ql/src/java/org/apache/hadoop/hive/ql/exec/vector/mapjoin/fast/VectorMapJoinFastHashTableLoader.java
   IDEA additional info:
   Subsystem: com.intellij.openapi.diff.impl.patch.CharsetEP
   <+>UTF-8
   ===================================================================
   diff --git 
a/ql/src/java/org/apache/hadoop/hive/ql/exec/vector/mapjoin/fast/VectorMapJoinFastHashTableLoader.java
 
b/ql/src/java/org/apache/hadoop/hive/ql/exec/vector/mapjoin/fast/VectorMapJoinFastHashTableLoader.java
   --- 
a/ql/src/java/org/apache/hadoop/hive/ql/exec/vector/mapjoin/fast/VectorMapJoinFastHashTableLoader.java
   (revision 7e356e06eee218e077586a52811a42449196152d)
   +++ 
b/ql/src/java/org/apache/hadoop/hive/ql/exec/vector/mapjoin/fast/VectorMapJoinFastHashTableLoader.java
   (date 1749805780973)
   @@ -88,10 +88,6 @@
    
      }
    
   -  ExecutorService getLoadExecService() {
   -    return loadExecService;
   -  }
   -
      public VectorMapJoinFastHashTableLoader(TezContext context, Configuration 
hconf, MapJoinOperator joinOp) {
        this.tezContext = context;
        this.hconf = hconf;
   @@ -114,7 +110,7 @@
        this.htLoadCounter = 
tezContext.getTezProcessorContext().getCounters().findCounter(counterGroup, 
counterName);
      }
    
   -   void initHTLoadingService(long estKeyCount) {
   +  void initHTLoadingService(long estKeyCount) {
        if (estKeyCount < VectorMapJoinFastHashTable.FIRST_SIZE_UP) {
          // Avoid many small HTs that will rehash multiple times causing GCs
          this.numLoadThreads = 1;
   @@ -157,8 +153,8 @@
        }
      }
    
   -   List<CompletableFuture<Void>> 
submitQueueDrainThreads(VectorMapJoinFastTableContainer 
vectorMapJoinFastTableContainer)
   -          throws InterruptedException, IOException, SerDeException {
   +  List<CompletableFuture<Void>> 
submitQueueDrainThreads(VectorMapJoinFastTableContainer 
vectorMapJoinFastTableContainer)
   +          throws InterruptedException {
        List<CompletableFuture<Void>> loaderTasks = new ArrayList<>();
        for (int partitionId = 0; partitionId < numLoadThreads; partitionId++) {
          int finalPartitionId = partitionId;
   @@ -319,20 +315,11 @@
    
            LOG.info("Finished loading the queue for input: {} waiting {} 
minutes for TPool shutdown", inputName, 2);
            addQueueDoneSentinel();
   -        loadExecService.shutdown();
            try {
              CompletableFuture.allOf(loaderTasks.toArray(new 
CompletableFuture[0]))
   -                  .get(2, TimeUnit.MINUTES);
   -        } catch (TimeoutException e) {
   -          throw new HiveException("Failed to complete the hash table 
loader. Loading timed out.");
   -        } catch (ExecutionException e) {
   -          throw new HiveException("One of the loader threads failed", 
e.getCause());
   -        } catch (InterruptedException e) {
   -          Thread.currentThread().interrupt();
   -        } finally {
   -          if (loadExecService != null && !loadExecService.isTerminated()) {
   -            loadExecService.shutdownNow();
   -          }
   +              .get(2, TimeUnit.MINUTES);
   +        } catch (ExecutionException | TimeoutException e) {
   +          throw new HiveException("Failed to complete the hash table 
loader.", e.getCause());
            }
            batchPool.clear();
            LOG.info("Total received entries: {} Threads {} HT entries: {}", 
receivedEntries, numLoadThreads, totalEntries.get());
   ````



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscr...@hive.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscr...@hive.apache.org
For additional commands, e-mail: gitbox-h...@hive.apache.org

Reply via email to