[ 
https://issues.apache.org/jira/browse/HIVE-23069?focusedWorklogId=459183&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-459183
 ]

ASF GitHub Bot logged work on HIVE-23069:
-----------------------------------------

                Author: ASF GitHub Bot
            Created on: 15/Jul/20 09:46
            Start Date: 15/Jul/20 09:46
    Worklog Time Spent: 10m 
      Work Description: pkumarsinha commented on a change in pull request #1225:
URL: https://github.com/apache/hive/pull/1225#discussion_r454928086



##########
File path: 
ql/src/java/org/apache/hadoop/hive/ql/exec/repl/util/FileListStreamer.java
##########
@@ -0,0 +1,150 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.hive.ql.exec.repl.util;
+
+import com.google.common.annotations.VisibleForTesting;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.FileSystem;
+import org.apache.hadoop.fs.Path;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.BufferedWriter;
+import java.io.Closeable;
+import java.io.IOException;
+import java.io.OutputStreamWriter;
+import java.util.concurrent.LinkedBlockingQueue;
+import java.util.concurrent.TimeUnit;
+
+public class FileListStreamer extends Thread implements Closeable {
+  private static final Logger LOG = 
LoggerFactory.getLogger(FileListStreamer.class);
+  private static BufferedWriter backingFileWriterInTest;
+  private static final long TIMEOUT_IN_SECS = 5L;
+  private volatile boolean signalTostop;
+  private final LinkedBlockingQueue<String> cache;
+  private Path backingFile;
+  private Configuration conf;
+  private BufferedWriter backingFileWriter;
+  private volatile boolean valid = true;
+  private final Object COMPLETION_LOCK = new Object();
+  private volatile boolean completed = false;
+  private volatile boolean initialized = false;
+
+
+
+  public FileListStreamer(LinkedBlockingQueue<String> cache, Path backingFile, 
Configuration conf) throws IOException {
+    this.cache = cache;
+    this.backingFile = backingFile;
+    this.conf = conf;
+  }
+
+  private void lazyInit() throws IOException {
+    if (backingFileWriterInTest == null) {
+      FileSystem fs = FileSystem.get(backingFile.toUri(), conf);
+      backingFileWriter = new BufferedWriter(new 
OutputStreamWriter(fs.create(backingFile)));
+    } else {
+      backingFileWriter = backingFileWriterInTest;
+    }
+    initialized = true;
+    LOG.info("Initialized a file based store to save a list at: {}", 
backingFile);
+  }
+
+  public boolean isValid() {
+    return valid;
+  }
+
+  // Blocks for remaining entries to be flushed to file.
+  @Override
+  public void close() throws IOException {
+    signalTostop = true;
+    synchronized (COMPLETION_LOCK) {
+      while (motiveToWait()) {
+        try {
+          COMPLETION_LOCK.wait(TimeUnit.SECONDS.toMillis(TIMEOUT_IN_SECS));
+        } catch (InterruptedException e) {
+          // no-op
+        }
+      }
+    }
+    if (!isValid()) {
+      throw new IOException("File list is not in a valid state:" + 
backingFile);
+    }
+  }
+
+  private boolean motiveToWait() {
+    return !completed && valid;
+  }
+
+  @Override
+  public void run() {
+    try {
+      lazyInit();
+    } catch (IOException e) {
+      valid = false;
+      throw new RuntimeException("Unable to initialize the file list 
streamer", e);
+    }
+    boolean exThrown = false;
+    while (!exThrown && (!signalTostop || !cache.isEmpty())) {
+      try {
+        String nextEntry = cache.poll(TIMEOUT_IN_SECS, TimeUnit.SECONDS);
+        if (nextEntry != null) {
+          backingFileWriter.write(nextEntry);
+          backingFileWriter.newLine();
+          LOG.debug("Writing entry {} to file list backed by {}", nextEntry, 
backingFile);
+        }
+      } catch (Exception iEx) {
+        if (!(iEx instanceof InterruptedException)) {
+          // not draining any more. Inform the producer to avoid OOM.
+          valid = false;
+          LOG.error("Exception while saving the list to file " + backingFile, 
iEx);
+          exThrown = true;
+        }
+      }
+    }
+    try{
+      closeBackingFile();
+      completed = true;
+    } finally {
+      synchronized (COMPLETION_LOCK) {
+        COMPLETION_LOCK.notify();
+      }
+    }
+    LOG.info("Completed the file list streamer backed by: {}", backingFile);
+  }
+
+  private void closeBackingFile() {
+    try {
+      backingFileWriter.close();
+      LOG.debug("Closed the file list backing file: {}", backingFile);
+    } catch (IOException e) {
+      LOG.error("Exception while closing the file list backing file", e);
+      valid = false;
+    }
+  }
+
+  @VisibleForTesting
+  public static void setBackingFileWriterInTest(BufferedWriter bufferedWriter) 
{

Review comment:
       How will thread work in mocked case?




----------------------------------------------------------------
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.

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


Issue Time Tracking
-------------------

    Worklog Id:     (was: 459183)
    Time Spent: 5h 10m  (was: 5h)

> Memory efficient iterator should be used during replication.
> ------------------------------------------------------------
>
>                 Key: HIVE-23069
>                 URL: https://issues.apache.org/jira/browse/HIVE-23069
>             Project: Hive
>          Issue Type: Improvement
>            Reporter: Pravin Sinha
>            Assignee: Pravin Sinha
>            Priority: Major
>              Labels: pull-request-available
>         Attachments: HIVE-23069.01.patch, HIVE-23069.02.patch
>
>          Time Spent: 5h 10m
>  Remaining Estimate: 0h
>
> Currently the iterator used while copying table data is memory based. In case 
> of a database with very large number of table/partitions, such iterator may 
> cause HS2 process to go OOM.
> Also introduces a config option to run data copy tasks during repl load 
> operation.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)

Reply via email to