001/** 002 * 003 * Licensed to the Apache Software Foundation (ASF) under one 004 * or more contributor license agreements. See the NOTICE file 005 * distributed with this work for additional information 006 * regarding copyright ownership. The ASF licenses this file 007 * to you under the Apache License, Version 2.0 (the 008 * "License"); you may not use this file except in compliance 009 * with the License. You may obtain a copy of the License at 010 * 011 * http://www.apache.org/licenses/LICENSE-2.0 012 * 013 * Unless required by applicable law or agreed to in writing, software 014 * distributed under the License is distributed on an "AS IS" BASIS, 015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 016 * See the License for the specific language governing permissions and 017 * limitations under the License. 018 */ 019package org.apache.hadoop.hbase.regionserver; 020 021import java.io.IOException; 022import java.util.Collection; 023import java.util.Comparator; 024import java.util.Iterator; 025import java.util.List; 026import java.util.Optional; 027import java.util.stream.Collectors; 028 029import org.apache.hadoop.conf.Configuration; 030import org.apache.hadoop.hbase.Cell; 031import org.apache.hadoop.hbase.CellComparator; 032import org.apache.hadoop.hbase.KeyValue; 033import org.apache.hadoop.hbase.regionserver.compactions.CompactionConfiguration; 034import org.apache.yetus.audience.InterfaceAudience; 035import org.slf4j.Logger; 036import org.slf4j.LoggerFactory; 037import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableCollection; 038import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableList; 039import org.apache.hbase.thirdparty.com.google.common.collect.Iterables; 040 041/** 042 * Default implementation of StoreFileManager. Not thread-safe. 043 */ 044@InterfaceAudience.Private 045class DefaultStoreFileManager implements StoreFileManager { 046 private static final Logger LOG = LoggerFactory.getLogger(DefaultStoreFileManager.class); 047 048 private final CellComparator cellComparator; 049 private final CompactionConfiguration comConf; 050 private final int blockingFileCount; 051 private final Comparator<HStoreFile> storeFileComparator; 052 /** 053 * List of store files inside this store. This is an immutable list that 054 * is atomically replaced when its contents change. 055 */ 056 private volatile ImmutableList<HStoreFile> storefiles = ImmutableList.of(); 057 /** 058 * List of compacted files inside this store that needs to be excluded in reads 059 * because further new reads will be using only the newly created files out of compaction. 060 * These compacted files will be deleted/cleared once all the existing readers on these 061 * compacted files are done. 062 */ 063 private volatile ImmutableList<HStoreFile> compactedfiles = ImmutableList.of(); 064 065 public DefaultStoreFileManager(CellComparator cellComparator, 066 Comparator<HStoreFile> storeFileComparator, Configuration conf, 067 CompactionConfiguration comConf) { 068 this.cellComparator = cellComparator; 069 this.storeFileComparator = storeFileComparator; 070 this.comConf = comConf; 071 this.blockingFileCount = 072 conf.getInt(HStore.BLOCKING_STOREFILES_KEY, HStore.DEFAULT_BLOCKING_STOREFILE_COUNT); 073 } 074 075 @Override 076 public void loadFiles(List<HStoreFile> storeFiles) { 077 this.storefiles = ImmutableList.sortedCopyOf(storeFileComparator, storeFiles); 078 } 079 080 @Override 081 public final Collection<HStoreFile> getStorefiles() { 082 return storefiles; 083 } 084 085 @Override 086 public Collection<HStoreFile> getCompactedfiles() { 087 return compactedfiles; 088 } 089 090 @Override 091 public void insertNewFiles(Collection<HStoreFile> sfs) throws IOException { 092 this.storefiles = 093 ImmutableList.sortedCopyOf(storeFileComparator, Iterables.concat(this.storefiles, sfs)); 094 } 095 096 @Override 097 public ImmutableCollection<HStoreFile> clearFiles() { 098 ImmutableList<HStoreFile> result = storefiles; 099 storefiles = ImmutableList.of(); 100 return result; 101 } 102 103 @Override 104 public Collection<HStoreFile> clearCompactedFiles() { 105 List<HStoreFile> result = compactedfiles; 106 compactedfiles = ImmutableList.of(); 107 return result; 108 } 109 110 @Override 111 public final int getStorefileCount() { 112 return storefiles.size(); 113 } 114 115 @Override 116 public final int getCompactedFilesCount() { 117 return compactedfiles.size(); 118 } 119 120 @Override 121 public void addCompactionResults(Collection<HStoreFile> newCompactedfiles, 122 Collection<HStoreFile> results) { 123 this.storefiles = ImmutableList.sortedCopyOf(storeFileComparator, Iterables 124 .concat(Iterables.filter(storefiles, sf -> !newCompactedfiles.contains(sf)), results)); 125 // Mark the files as compactedAway once the storefiles and compactedfiles list is finalized 126 // Let a background thread close the actual reader on these compacted files and also 127 // ensure to evict the blocks from block cache so that they are no longer in 128 // cache 129 newCompactedfiles.forEach(HStoreFile::markCompactedAway); 130 this.compactedfiles = ImmutableList.sortedCopyOf(storeFileComparator, 131 Iterables.concat(this.compactedfiles, newCompactedfiles)); 132 } 133 134 @Override 135 public void removeCompactedFiles(Collection<HStoreFile> removedCompactedfiles) 136 throws IOException { 137 this.compactedfiles = 138 this.compactedfiles.stream().filter(sf -> !removedCompactedfiles.contains(sf)) 139 .sorted(storeFileComparator).collect(ImmutableList.toImmutableList()); 140 } 141 142 @Override 143 public final Iterator<HStoreFile> getCandidateFilesForRowKeyBefore(KeyValue targetKey) { 144 return this.storefiles.reverse().iterator(); 145 } 146 147 @Override 148 public Iterator<HStoreFile> updateCandidateFilesForRowKeyBefore( 149 Iterator<HStoreFile> candidateFiles, KeyValue targetKey, Cell candidate) { 150 // Default store has nothing useful to do here. 151 // TODO: move this comment when implementing Level: 152 // Level store can trim the list by range, removing all the files which cannot have 153 // any useful candidates less than "candidate". 154 return candidateFiles; 155 } 156 157 @Override 158 public final Optional<byte[]> getSplitPoint() throws IOException { 159 return StoreUtils.getSplitPoint(storefiles, cellComparator); 160 } 161 162 @Override 163 public final Collection<HStoreFile> getFilesForScan(byte[] startRow, boolean includeStartRow, 164 byte[] stopRow, boolean includeStopRow) { 165 // We cannot provide any useful input and already have the files sorted by seqNum. 166 return getStorefiles(); 167 } 168 169 @Override 170 public int getStoreCompactionPriority() { 171 int priority = blockingFileCount - storefiles.size(); 172 return (priority == HStore.PRIORITY_USER) ? priority + 1 : priority; 173 } 174 175 @Override 176 public Collection<HStoreFile> getUnneededFiles(long maxTs, List<HStoreFile> filesCompacting) { 177 ImmutableList<HStoreFile> files = storefiles; 178 // 1) We can never get rid of the last file which has the maximum seqid. 179 // 2) Files that are not the latest can't become one due to (1), so the rest are fair game. 180 return files.stream().limit(Math.max(0, files.size() - 1)).filter(sf -> { 181 long fileTs = sf.getReader().getMaxTimestamp(); 182 if (fileTs < maxTs && !filesCompacting.contains(sf)) { 183 LOG.info("Found an expired store file {} whose maxTimestamp is {}, which is below {}", 184 sf.getPath(), fileTs, maxTs); 185 return true; 186 } else { 187 return false; 188 } 189 }).collect(Collectors.toList()); 190 } 191 192 @Override 193 public double getCompactionPressure() { 194 int storefileCount = getStorefileCount(); 195 int minFilesToCompact = comConf.getMinFilesToCompact(); 196 if (storefileCount <= minFilesToCompact) { 197 return 0.0; 198 } 199 return (double) (storefileCount - minFilesToCompact) / (blockingFileCount - minFilesToCompact); 200 } 201 202 @Override 203 public Comparator<HStoreFile> getStoreFileComparator() { 204 return storeFileComparator; 205 } 206} 207