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.util.compaction;
019
020import java.io.IOException;
021import java.util.Collection;
022import java.util.List;
023import java.util.Optional;
024import java.util.Set;
025import org.apache.hadoop.conf.Configuration;
026import org.apache.hadoop.fs.FileStatus;
027import org.apache.hadoop.fs.FileSystem;
028import org.apache.hadoop.fs.Path;
029import org.apache.hadoop.hbase.client.Admin;
030import org.apache.hadoop.hbase.client.Connection;
031import org.apache.hadoop.hbase.client.ConnectionFactory;
032import org.apache.hadoop.hbase.client.RegionInfo;
033import org.apache.hadoop.hbase.regionserver.HRegionFileSystem;
034import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
035import org.apache.hadoop.hbase.util.CommonFSUtils;
036import org.apache.hadoop.hbase.util.FSUtils;
037import org.apache.yetus.audience.InterfaceAudience;
038import org.slf4j.Logger;
039import org.slf4j.LoggerFactory;
040
041import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
042import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
043
044@InterfaceAudience.Private
045class MajorCompactionRequest {
046
047  private static final Logger LOG = LoggerFactory.getLogger(MajorCompactionRequest.class);
048
049  protected final Configuration configuration;
050  protected final RegionInfo region;
051  private Set<String> stores;
052
053  MajorCompactionRequest(Configuration configuration, RegionInfo region) {
054    this.configuration = configuration;
055    this.region = region;
056  }
057
058  @VisibleForTesting
059  MajorCompactionRequest(Configuration configuration, RegionInfo region,
060      Set<String> stores) {
061    this(configuration, region);
062    this.stores = stores;
063  }
064
065  static Optional<MajorCompactionRequest> newRequest(Configuration configuration, RegionInfo info,
066      Set<String> stores, long timestamp) throws IOException {
067    MajorCompactionRequest request =
068        new MajorCompactionRequest(configuration, info, stores);
069    return request.createRequest(configuration, stores, timestamp);
070  }
071
072  RegionInfo getRegion() {
073    return region;
074  }
075
076  Set<String> getStores() {
077    return stores;
078  }
079
080  void setStores(Set<String> stores) {
081    this.stores = stores;
082  }
083
084  @VisibleForTesting
085  Optional<MajorCompactionRequest> createRequest(Configuration configuration,
086      Set<String> stores, long timestamp) throws IOException {
087    Set<String> familiesToCompact = getStoresRequiringCompaction(stores, timestamp);
088    MajorCompactionRequest request = null;
089    if (!familiesToCompact.isEmpty()) {
090      request = new MajorCompactionRequest(configuration, region, familiesToCompact);
091    }
092    return Optional.ofNullable(request);
093  }
094
095  Set<String> getStoresRequiringCompaction(Set<String> requestedStores, long timestamp)
096      throws IOException {
097    try(Connection connection = getConnection(configuration)) {
098      HRegionFileSystem fileSystem = getFileSystem(connection);
099      Set<String> familiesToCompact = Sets.newHashSet();
100      for (String family : requestedStores) {
101        if (shouldCFBeCompacted(fileSystem, family, timestamp)) {
102          familiesToCompact.add(family);
103        }
104      }
105      return familiesToCompact;
106    }
107  }
108
109  boolean shouldCFBeCompacted(HRegionFileSystem fileSystem, String family, long ts)
110      throws IOException {
111
112    // do we have any store files?
113    Collection<StoreFileInfo> storeFiles = fileSystem.getStoreFiles(family);
114    if (storeFiles == null) {
115      LOG.info("Excluding store: " + family + " for compaction for region:  " + fileSystem
116          .getRegionInfo().getEncodedName(), " has no store files");
117      return false;
118    }
119    // check for reference files
120    if (fileSystem.hasReferences(family) && familyHasReferenceFile(fileSystem, family, ts)) {
121      LOG.info("Including store: " + family + " with: " + storeFiles.size()
122          + " files for compaction for region: " + fileSystem.getRegionInfo().getEncodedName());
123      return true;
124    }
125    // check store file timestamps
126    boolean includeStore = this.shouldIncludeStore(fileSystem, family, storeFiles, ts);
127    if (!includeStore) {
128      LOG.info("Excluding store: " + family + " for compaction for region:  " + fileSystem
129          .getRegionInfo().getEncodedName() + " already compacted");
130    }
131    return includeStore;
132  }
133
134  protected boolean shouldIncludeStore(HRegionFileSystem fileSystem, String family,
135      Collection<StoreFileInfo> storeFiles, long ts) throws IOException {
136
137    for (StoreFileInfo storeFile : storeFiles) {
138      if (storeFile.getModificationTime() < ts) {
139        LOG.info("Including store: " + family + " with: " + storeFiles.size()
140            + " files for compaction for region: "
141            + fileSystem.getRegionInfo().getEncodedName());
142        return true;
143      }
144    }
145    return false;
146  }
147
148  @VisibleForTesting
149  Connection getConnection(Configuration configuration) throws IOException {
150    return ConnectionFactory.createConnection(configuration);
151  }
152
153  protected boolean familyHasReferenceFile(HRegionFileSystem fileSystem, String family, long ts)
154      throws IOException {
155    List<Path> referenceFiles =
156        getReferenceFilePaths(fileSystem.getFileSystem(), fileSystem.getStoreDir(family));
157    for (Path referenceFile : referenceFiles) {
158      FileStatus status = fileSystem.getFileSystem().getFileLinkStatus(referenceFile);
159      if (status.getModificationTime() < ts) {
160        LOG.info("Including store: " + family + " for compaction for region:  " + fileSystem
161            .getRegionInfo().getEncodedName() + " (reference store files)");
162        return true;
163      }
164    }
165    return false;
166
167  }
168
169  @VisibleForTesting
170  List<Path> getReferenceFilePaths(FileSystem fileSystem, Path familyDir)
171      throws IOException {
172    return FSUtils.getReferenceFilePaths(fileSystem, familyDir);
173  }
174
175  @VisibleForTesting
176  HRegionFileSystem getFileSystem(Connection connection) throws IOException {
177    Admin admin = connection.getAdmin();
178    return HRegionFileSystem.openRegionFromFileSystem(admin.getConfiguration(),
179      CommonFSUtils.getCurrentFileSystem(admin.getConfiguration()), CommonFSUtils.getTableDir(
180        CommonFSUtils.getRootDir(admin.getConfiguration()), region.getTable()),
181      region, true);
182  }
183
184  @Override
185  public String toString() {
186    return "region: " + region.getEncodedName() + " store(s): " + stores;
187  }
188}