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