001/* 002 * Licensed to the Apache Software Foundation (ASF) under one 003 * or more contributor license agreements. See the NOTICE file 004 * distributed with this work for additional information 005 * regarding copyright ownership. The ASF licenses this file 006 * to you under the Apache License, Version 2.0 (the 007 * "License"); you may not use this file except in compliance 008 * with the License. You may obtain a copy of the License at 009 * 010 * http://www.apache.org/licenses/LICENSE-2.0 011 * 012 * Unless required by applicable law or agreed to in writing, software 013 * distributed under the License is distributed on an "AS IS" BASIS, 014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 015 * See the License for the specific language governing permissions and 016 * limitations under the License. 017 */ 018package org.apache.hadoop.hbase.master.cleaner; 019 020import static org.apache.hadoop.hbase.HConstants.HBASE_MASTER_LOGCLEANER_PLUGINS; 021 022import java.io.IOException; 023import java.util.ArrayList; 024import java.util.List; 025import java.util.Map; 026import java.util.concurrent.CountDownLatch; 027import java.util.concurrent.LinkedBlockingQueue; 028import java.util.concurrent.TimeUnit; 029import java.util.concurrent.atomic.AtomicBoolean; 030import java.util.stream.Collectors; 031import org.apache.hadoop.conf.Configuration; 032import org.apache.hadoop.fs.FileStatus; 033import org.apache.hadoop.fs.FileSystem; 034import org.apache.hadoop.fs.Path; 035import org.apache.hadoop.hbase.Stoppable; 036import org.apache.hadoop.hbase.conf.ConfigurationObserver; 037import org.apache.hadoop.hbase.master.procedure.MasterProcedureUtil; 038import org.apache.hadoop.hbase.master.region.MasterRegionFactory; 039import org.apache.hadoop.hbase.wal.AbstractFSWALProvider; 040import org.apache.yetus.audience.InterfaceAudience; 041import org.slf4j.Logger; 042import org.slf4j.LoggerFactory; 043 044import org.apache.hbase.thirdparty.com.google.common.base.Preconditions; 045 046/** 047 * This Chore, every time it runs, will attempt to delete the WALs and Procedure WALs in the old 048 * logs folder. The WAL is only deleted if none of the cleaner delegates says otherwise. 049 * @see BaseLogCleanerDelegate 050 */ 051@InterfaceAudience.Private 052public class LogCleaner extends CleanerChore<BaseLogCleanerDelegate> 053 implements ConfigurationObserver { 054 private static final Logger LOG = LoggerFactory.getLogger(LogCleaner.class); 055 056 public static final String OLD_WALS_CLEANER_THREAD_SIZE = "hbase.oldwals.cleaner.thread.size"; 057 public static final int DEFAULT_OLD_WALS_CLEANER_THREAD_SIZE = 2; 058 059 public static final String OLD_WALS_CLEANER_THREAD_TIMEOUT_MSEC = 060 "hbase.oldwals.cleaner.thread.timeout.msec"; 061 static final long DEFAULT_OLD_WALS_CLEANER_THREAD_TIMEOUT_MSEC = 60 * 1000L; 062 063 private final LinkedBlockingQueue<CleanerContext> pendingDelete; 064 private List<Thread> oldWALsCleaner; 065 private long cleanerThreadTimeoutMsec; 066 067 /** 068 * @param period the period of time to sleep between each run 069 * @param stopper the stopper 070 * @param conf configuration to use 071 * @param fs handle to the FS 072 * @param oldLogDir the path to the archived logs 073 * @param pool the thread pool used to scan directories 074 */ 075 public LogCleaner(final int period, final Stoppable stopper, Configuration conf, FileSystem fs, 076 Path oldLogDir, DirScanPool pool, Map<String, Object> params) { 077 super("LogsCleaner", period, stopper, conf, fs, oldLogDir, HBASE_MASTER_LOGCLEANER_PLUGINS, 078 pool, params, null); 079 this.pendingDelete = new LinkedBlockingQueue<>(); 080 int size = conf.getInt(OLD_WALS_CLEANER_THREAD_SIZE, DEFAULT_OLD_WALS_CLEANER_THREAD_SIZE); 081 this.oldWALsCleaner = createOldWalsCleaner(size); 082 this.cleanerThreadTimeoutMsec = conf.getLong(OLD_WALS_CLEANER_THREAD_TIMEOUT_MSEC, 083 DEFAULT_OLD_WALS_CLEANER_THREAD_TIMEOUT_MSEC); 084 } 085 086 @Override 087 protected boolean validate(Path file) { 088 return AbstractFSWALProvider.validateWALFilename(file.getName()) 089 || MasterProcedureUtil.validateProcedureWALFilename(file.getName()) 090 || file.getName().endsWith(MasterRegionFactory.ARCHIVED_WAL_SUFFIX); 091 } 092 093 @Override 094 public void onConfigurationChange(Configuration conf) { 095 int newSize = conf.getInt(OLD_WALS_CLEANER_THREAD_SIZE, DEFAULT_OLD_WALS_CLEANER_THREAD_SIZE); 096 if (newSize == oldWALsCleaner.size()) { 097 LOG.debug( 098 "Size from configuration is the same as previous which " + "is {}, no need to update.", 099 newSize); 100 return; 101 } 102 interruptOldWALsCleaner(); 103 oldWALsCleaner = createOldWalsCleaner(newSize); 104 cleanerThreadTimeoutMsec = conf.getLong(OLD_WALS_CLEANER_THREAD_TIMEOUT_MSEC, 105 DEFAULT_OLD_WALS_CLEANER_THREAD_TIMEOUT_MSEC); 106 } 107 108 @Override 109 protected int deleteFiles(Iterable<FileStatus> filesToDelete) { 110 List<CleanerContext> results = new ArrayList<>(); 111 for (FileStatus file : filesToDelete) { 112 LOG.trace("Scheduling file {} for deletion", file); 113 if (file != null) { 114 results.add(new CleanerContext(file)); 115 } 116 } 117 if (results.isEmpty()) { 118 return 0; 119 } 120 121 LOG.debug("Old WALs for delete: {}", 122 results.stream().map(cc -> cc.target.getPath().getName()).collect(Collectors.joining(", "))); 123 pendingDelete.addAll(results); 124 125 int deletedFiles = 0; 126 for (CleanerContext res : results) { 127 LOG.trace("Awaiting the results for deletion of old WAL file: {}", res); 128 deletedFiles += res.getResult(this.cleanerThreadTimeoutMsec) ? 1 : 0; 129 } 130 return deletedFiles; 131 } 132 133 @Override 134 public synchronized void cleanup() { 135 super.cleanup(); 136 interruptOldWALsCleaner(); 137 } 138 139 int getSizeOfCleaners() { 140 return oldWALsCleaner.size(); 141 } 142 143 long getCleanerThreadTimeoutMsec() { 144 return cleanerThreadTimeoutMsec; 145 } 146 147 private List<Thread> createOldWalsCleaner(int size) { 148 LOG.info("Creating {} old WALs cleaner threads", size); 149 150 List<Thread> oldWALsCleaner = new ArrayList<>(size); 151 for (int i = 0; i < size; i++) { 152 Thread cleaner = new Thread(() -> deleteFile()); 153 cleaner.setName("OldWALsCleaner-" + i); 154 cleaner.setDaemon(true); 155 cleaner.start(); 156 oldWALsCleaner.add(cleaner); 157 } 158 return oldWALsCleaner; 159 } 160 161 private void interruptOldWALsCleaner() { 162 for (Thread cleaner : oldWALsCleaner) { 163 LOG.trace("Interrupting thread: {}", cleaner); 164 cleaner.interrupt(); 165 } 166 oldWALsCleaner.clear(); 167 } 168 169 private void deleteFile() { 170 while (true) { 171 try { 172 final CleanerContext context = pendingDelete.take(); 173 Preconditions.checkNotNull(context); 174 FileStatus oldWalFile = context.getTargetToClean(); 175 try { 176 LOG.debug("Deleting {}", oldWalFile); 177 boolean succeed = this.fs.delete(oldWalFile.getPath(), false); 178 context.setResult(succeed); 179 } catch (IOException e) { 180 // fs.delete() fails. 181 LOG.warn("Failed to delete old WAL file", e); 182 context.setResult(false); 183 } 184 } catch (InterruptedException ite) { 185 // It is most likely from configuration changing request 186 LOG.warn( 187 "Interrupted while cleaning old WALs, will " + "try to clean it next round. Exiting."); 188 // Restore interrupt status 189 Thread.currentThread().interrupt(); 190 return; 191 } 192 LOG.trace("Exiting"); 193 } 194 } 195 196 @Override 197 public synchronized void cancel(boolean mayInterruptIfRunning) { 198 LOG.debug("Cancelling LogCleaner"); 199 super.cancel(mayInterruptIfRunning); 200 interruptOldWALsCleaner(); 201 } 202 203 private static final class CleanerContext { 204 205 final FileStatus target; 206 final AtomicBoolean result; 207 final CountDownLatch remainingResults; 208 209 private CleanerContext(FileStatus status) { 210 this.target = status; 211 this.result = new AtomicBoolean(false); 212 this.remainingResults = new CountDownLatch(1); 213 } 214 215 void setResult(boolean res) { 216 this.result.set(res); 217 this.remainingResults.countDown(); 218 } 219 220 boolean getResult(long waitIfNotFinished) { 221 try { 222 boolean completed = this.remainingResults.await(waitIfNotFinished, TimeUnit.MILLISECONDS); 223 if (!completed) { 224 LOG.warn("Spent too much time [{}ms] deleting old WAL file: {}", waitIfNotFinished, 225 target); 226 return false; 227 } 228 } catch (InterruptedException e) { 229 LOG.warn("Interrupted while awaiting deletion of WAL file: {}", target); 230 return false; 231 } 232 return result.get(); 233 } 234 235 FileStatus getTargetToClean() { 236 return target; 237 } 238 239 @Override 240 public String toString() { 241 return "CleanerContext [target=" + target + ", result=" + result + "]"; 242 } 243 } 244}