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.regionserver;
019
020import static org.junit.Assert.assertNotNull;
021import static org.junit.Assert.assertTrue;
022
023import java.io.IOException;
024import java.security.Key;
025import java.util.ArrayList;
026import java.util.Collection;
027import java.util.List;
028import javax.crypto.spec.SecretKeySpec;
029import org.apache.hadoop.conf.Configuration;
030import org.apache.hadoop.fs.Path;
031import org.apache.hadoop.hbase.HBaseClassTestRule;
032import org.apache.hadoop.hbase.HBaseTestingUtil;
033import org.apache.hadoop.hbase.HConstants;
034import org.apache.hadoop.hbase.TableName;
035import org.apache.hadoop.hbase.Waiter;
036import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
037import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder;
038import org.apache.hadoop.hbase.client.CompactionState;
039import org.apache.hadoop.hbase.client.Put;
040import org.apache.hadoop.hbase.client.Table;
041import org.apache.hadoop.hbase.client.TableDescriptor;
042import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
043import org.apache.hadoop.hbase.io.crypto.Encryption;
044import org.apache.hadoop.hbase.io.crypto.KeyProviderForTesting;
045import org.apache.hadoop.hbase.io.crypto.aes.AES;
046import org.apache.hadoop.hbase.io.hfile.CacheConfig;
047import org.apache.hadoop.hbase.io.hfile.HFile;
048import org.apache.hadoop.hbase.security.EncryptionUtil;
049import org.apache.hadoop.hbase.security.User;
050import org.apache.hadoop.hbase.testclassification.MediumTests;
051import org.apache.hadoop.hbase.testclassification.RegionServerTests;
052import org.apache.hadoop.hbase.util.Bytes;
053import org.junit.AfterClass;
054import org.junit.BeforeClass;
055import org.junit.ClassRule;
056import org.junit.Rule;
057import org.junit.Test;
058import org.junit.experimental.categories.Category;
059import org.junit.rules.TestName;
060
061@Category({ RegionServerTests.class, MediumTests.class })
062public class TestEncryptionKeyRotation {
063
064  @ClassRule
065  public static final HBaseClassTestRule CLASS_RULE =
066    HBaseClassTestRule.forClass(TestEncryptionKeyRotation.class);
067
068  private static final HBaseTestingUtil TEST_UTIL = new HBaseTestingUtil();
069  private static final Configuration conf = TEST_UTIL.getConfiguration();
070  private static final Key initialCFKey;
071  private static final Key secondCFKey;
072
073  @Rule
074  public TestName name = new TestName();
075
076  static {
077    // Create the test encryption keys
078    byte[] keyBytes = new byte[AES.KEY_LENGTH];
079    Bytes.secureRandom(keyBytes);
080    String algorithm = conf.get(HConstants.CRYPTO_KEY_ALGORITHM_CONF_KEY, HConstants.CIPHER_AES);
081    initialCFKey = new SecretKeySpec(keyBytes, algorithm);
082    Bytes.secureRandom(keyBytes);
083    secondCFKey = new SecretKeySpec(keyBytes, algorithm);
084  }
085
086  @BeforeClass
087  public static void setUp() throws Exception {
088    conf.setInt("hfile.format.version", 3);
089    conf.set(HConstants.CRYPTO_KEYPROVIDER_CONF_KEY, KeyProviderForTesting.class.getName());
090    conf.set(HConstants.CRYPTO_MASTERKEY_NAME_CONF_KEY, "hbase");
091
092    // Start the minicluster
093    TEST_UTIL.startMiniCluster(1);
094  }
095
096  @AfterClass
097  public static void tearDown() throws Exception {
098    TEST_UTIL.shutdownMiniCluster();
099  }
100
101  @Test
102  public void testCFKeyRotation() throws Exception {
103    // Create the table schema
104    TableDescriptorBuilder tableDescriptorBuilder =
105      TableDescriptorBuilder.newBuilder(TableName.valueOf("default", name.getMethodName()));
106    ColumnFamilyDescriptorBuilder columnFamilyDescriptorBuilder =
107      ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes("cf"));
108    String algorithm = conf.get(HConstants.CRYPTO_KEY_ALGORITHM_CONF_KEY, HConstants.CIPHER_AES);
109    columnFamilyDescriptorBuilder.setEncryptionType(algorithm);
110    columnFamilyDescriptorBuilder
111      .setEncryptionKey(EncryptionUtil.wrapKey(conf, "hbase", initialCFKey));
112    tableDescriptorBuilder.setColumnFamily(columnFamilyDescriptorBuilder.build());
113    TableDescriptor tableDescriptor = tableDescriptorBuilder.build();
114
115    // Create the table and some on disk files
116    createTableAndFlush(tableDescriptor);
117
118    // Verify we have store file(s) with the initial key
119    final List<Path> initialPaths = findStorefilePaths(tableDescriptor.getTableName());
120    assertTrue(initialPaths.size() > 0);
121    for (Path path : initialPaths) {
122      assertTrue("Store file " + path + " has incorrect key",
123        Bytes.equals(initialCFKey.getEncoded(), extractHFileKey(path)));
124    }
125
126    // Update the schema with a new encryption key
127    columnFamilyDescriptorBuilder.setEncryptionKey(EncryptionUtil.wrapKey(conf,
128      conf.get(HConstants.CRYPTO_MASTERKEY_NAME_CONF_KEY, User.getCurrent().getShortName()),
129      secondCFKey));
130    TEST_UTIL.getAdmin().modifyColumnFamily(tableDescriptor.getTableName(),
131      columnFamilyDescriptorBuilder.build());
132    Thread.sleep(5000); // Need a predicate for online schema change
133
134    // And major compact
135    TEST_UTIL.getAdmin().majorCompact(tableDescriptor.getTableName());
136    // waiting for the major compaction to complete
137    TEST_UTIL.waitFor(30000, new Waiter.Predicate<IOException>() {
138      @Override
139      public boolean evaluate() throws IOException {
140        return TEST_UTIL.getAdmin().getCompactionState(tableDescriptor.getTableName())
141            == CompactionState.NONE;
142      }
143    });
144    List<Path> pathsAfterCompaction = findStorefilePaths(tableDescriptor.getTableName());
145    assertTrue(pathsAfterCompaction.size() > 0);
146    for (Path path : pathsAfterCompaction) {
147      assertTrue("Store file " + path + " has incorrect key",
148        Bytes.equals(secondCFKey.getEncoded(), extractHFileKey(path)));
149    }
150    List<Path> compactedPaths = findCompactedStorefilePaths(tableDescriptor.getTableName());
151    assertTrue(compactedPaths.size() > 0);
152    for (Path path : compactedPaths) {
153      assertTrue("Store file " + path + " retains initial key",
154        Bytes.equals(initialCFKey.getEncoded(), extractHFileKey(path)));
155    }
156  }
157
158  @Test
159  public void testMasterKeyRotation() throws Exception {
160    // Create the table schema
161    TableDescriptorBuilder tableDescriptorBuilder =
162      TableDescriptorBuilder.newBuilder(TableName.valueOf("default", name.getMethodName()));
163    ColumnFamilyDescriptorBuilder columnFamilyDescriptorBuilder =
164      ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes("cf"));
165    String algorithm = conf.get(HConstants.CRYPTO_KEY_ALGORITHM_CONF_KEY, HConstants.CIPHER_AES);
166    columnFamilyDescriptorBuilder.setEncryptionType(algorithm);
167    columnFamilyDescriptorBuilder
168      .setEncryptionKey(EncryptionUtil.wrapKey(conf, "hbase", initialCFKey));
169    tableDescriptorBuilder.setColumnFamily(columnFamilyDescriptorBuilder.build());
170    TableDescriptor tableDescriptor = tableDescriptorBuilder.build();
171
172    // Create the table and some on disk files
173    createTableAndFlush(tableDescriptor);
174
175    // Verify we have store file(s) with the initial key
176    List<Path> storeFilePaths = findStorefilePaths(tableDescriptor.getTableName());
177    assertTrue(storeFilePaths.size() > 0);
178    for (Path path : storeFilePaths) {
179      assertTrue("Store file " + path + " has incorrect key",
180        Bytes.equals(initialCFKey.getEncoded(), extractHFileKey(path)));
181    }
182
183    // Now shut down the HBase cluster
184    TEST_UTIL.shutdownMiniHBaseCluster();
185
186    // "Rotate" the master key
187    conf.set(HConstants.CRYPTO_MASTERKEY_NAME_CONF_KEY, "other");
188    conf.set(HConstants.CRYPTO_MASTERKEY_ALTERNATE_NAME_CONF_KEY, "hbase");
189
190    // Start the cluster back up
191    TEST_UTIL.startMiniHBaseCluster();
192    // Verify the table can still be loaded
193    TEST_UTIL.waitTableAvailable(tableDescriptor.getTableName(), 5000);
194    // Double check that the store file keys can be unwrapped
195    storeFilePaths = findStorefilePaths(tableDescriptor.getTableName());
196    assertTrue(storeFilePaths.size() > 0);
197    for (Path path : storeFilePaths) {
198      assertTrue("Store file " + path + " has incorrect key",
199        Bytes.equals(initialCFKey.getEncoded(), extractHFileKey(path)));
200    }
201  }
202
203  private static List<Path> findStorefilePaths(TableName tableName) throws Exception {
204    List<Path> paths = new ArrayList<>();
205    for (Region region : TEST_UTIL.getRSForFirstRegionInTable(tableName).getRegions(tableName)) {
206      for (HStore store : ((HRegion) region).getStores()) {
207        for (HStoreFile storefile : store.getStorefiles()) {
208          paths.add(storefile.getPath());
209        }
210      }
211    }
212    return paths;
213  }
214
215  private static List<Path> findCompactedStorefilePaths(TableName tableName) throws Exception {
216    List<Path> paths = new ArrayList<>();
217    for (Region region : TEST_UTIL.getRSForFirstRegionInTable(tableName).getRegions(tableName)) {
218      for (HStore store : ((HRegion) region).getStores()) {
219        Collection<HStoreFile> compactedfiles =
220          store.getStoreEngine().getStoreFileManager().getCompactedfiles();
221        if (compactedfiles != null) {
222          for (HStoreFile storefile : compactedfiles) {
223            paths.add(storefile.getPath());
224          }
225        }
226      }
227    }
228    return paths;
229  }
230
231  private void createTableAndFlush(TableDescriptor tableDescriptor) throws Exception {
232    ColumnFamilyDescriptor cfd = tableDescriptor.getColumnFamilies()[0];
233    // Create the test table
234    TEST_UTIL.getAdmin().createTable(tableDescriptor);
235    TEST_UTIL.waitTableAvailable(tableDescriptor.getTableName(), 5000);
236    // Create a store file
237    Table table = TEST_UTIL.getConnection().getTable(tableDescriptor.getTableName());
238    try {
239      table.put(new Put(Bytes.toBytes("testrow")).addColumn(cfd.getName(), Bytes.toBytes("q"),
240        Bytes.toBytes("value")));
241    } finally {
242      table.close();
243    }
244    TEST_UTIL.getAdmin().flush(tableDescriptor.getTableName());
245  }
246
247  private static byte[] extractHFileKey(Path path) throws Exception {
248    HFile.Reader reader =
249      HFile.createReader(TEST_UTIL.getTestFileSystem(), path, new CacheConfig(conf), true, conf);
250    try {
251      Encryption.Context cryptoContext = reader.getFileContext().getEncryptionContext();
252      assertNotNull("Reader has a null crypto context", cryptoContext);
253      Key key = cryptoContext.getKey();
254      assertNotNull("Crypto context has no key", key);
255      return key.getEncoded();
256    } finally {
257      reader.close();
258    }
259  }
260
261}