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.procedure;
019
020import static org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.EnableTableState.ENABLE_TABLE_MARK_REGIONS_ONLINE;
021import static org.junit.jupiter.api.Assertions.assertEquals;
022import static org.junit.jupiter.api.Assertions.assertFalse;
023import static org.junit.jupiter.api.Assertions.fail;
024
025import java.io.IOException;
026import java.util.Arrays;
027import java.util.Comparator;
028import java.util.List;
029import java.util.stream.Collectors;
030import org.apache.hadoop.hbase.TableName;
031import org.apache.hadoop.hbase.client.SnapshotDescription;
032import org.apache.hadoop.hbase.client.SnapshotType;
033import org.apache.hadoop.hbase.master.snapshot.SnapshotManager;
034import org.apache.hadoop.hbase.procedure2.Procedure;
035import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
036import org.apache.hadoop.hbase.procedure2.ProcedureTestingUtility;
037import org.apache.hadoop.hbase.snapshot.SnapshotTestingUtils;
038import org.apache.hadoop.hbase.testclassification.LargeTests;
039import org.apache.hadoop.hbase.testclassification.MasterTests;
040import org.junit.jupiter.api.Tag;
041import org.junit.jupiter.api.Test;
042
043import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
044import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
045import org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos;
046
047@Tag(MasterTests.TAG)
048@Tag(LargeTests.TAG)
049public class TestSnapshotProcedureConcurrently extends TestSnapshotProcedure {
050
051  @Test
052  public void testRunningTwoSnapshotProcedureOnSameTable() throws Exception {
053    String newSnapshotName = SNAPSHOT_NAME + "_2";
054    SnapshotProtos.SnapshotDescription snapshotProto2 =
055      SnapshotProtos.SnapshotDescription.newBuilder(snapshotProto).setName(newSnapshotName).build();
056
057    ProcedureExecutor<MasterProcedureEnv> procExec = master.getMasterProcedureExecutor();
058    MasterProcedureEnv env = procExec.getEnvironment();
059
060    SnapshotProcedure sp1 = new SnapshotProcedure(env, snapshotProto);
061    SnapshotProcedure sp2 = new SnapshotProcedure(env, snapshotProto2);
062    SnapshotProcedure spySp1 =
063      getDelayedOnSpecificStateSnapshotProcedure(sp1, procExec.getEnvironment(),
064        MasterProcedureProtos.SnapshotState.SNAPSHOT_SNAPSHOT_ONLINE_REGIONS);
065    SnapshotProcedure spySp2 =
066      getDelayedOnSpecificStateSnapshotProcedure(sp2, procExec.getEnvironment(),
067        MasterProcedureProtos.SnapshotState.SNAPSHOT_SNAPSHOT_ONLINE_REGIONS);
068
069    long procId1 = procExec.submitProcedure(spySp1);
070    long procId2 = procExec.submitProcedure(spySp2);
071    TEST_UTIL.waitFor(2000,
072      () -> env.getMasterServices().getProcedures().stream().map(Procedure::getProcId)
073        .collect(Collectors.toList()).containsAll(Arrays.asList(procId1, procId2)));
074
075    assertFalse(procExec.isFinished(procId1));
076    assertFalse(procExec.isFinished(procId2));
077
078    ProcedureTestingUtility.waitProcedure(master.getMasterProcedureExecutor(), procId1);
079    ProcedureTestingUtility.waitProcedure(master.getMasterProcedureExecutor(), procId2);
080
081    List<SnapshotProtos.SnapshotDescription> snapshots =
082      master.getSnapshotManager().getCompletedSnapshots();
083    assertEquals(2, snapshots.size());
084    snapshots.sort(Comparator.comparing(SnapshotProtos.SnapshotDescription::getName));
085    assertEquals(SNAPSHOT_NAME, snapshots.get(0).getName());
086    assertEquals(newSnapshotName, snapshots.get(1).getName());
087    SnapshotTestingUtils.confirmSnapshotValid(TEST_UTIL, snapshotProto, TABLE_NAME, CF);
088    SnapshotTestingUtils.confirmSnapshotValid(TEST_UTIL, snapshotProto2, TABLE_NAME, CF);
089  }
090
091  @Test
092  public void testTakeZkCoordinatedSnapshotAndProcedureCoordinatedSnapshotBoth() throws Exception {
093    String newSnapshotName = SNAPSHOT_NAME + "_2";
094    Thread first = new Thread("procedure-snapshot") {
095      @Override
096      public void run() {
097        try {
098          TEST_UTIL.getAdmin().snapshot(snapshot);
099        } catch (IOException e) {
100          LOG.error("procedure snapshot failed", e);
101          fail("procedure snapshot failed");
102        }
103      }
104    };
105    first.start();
106    Thread.sleep(1000);
107
108    SnapshotManager sm = master.getSnapshotManager();
109    TEST_UTIL.waitFor(2000, 50,
110      () -> !sm.isTakingSnapshot(TABLE_NAME) && sm.isTableTakingAnySnapshot(TABLE_NAME));
111
112    TEST_UTIL.getConfiguration().setBoolean("hbase.snapshot.zk.coordinated", true);
113    SnapshotDescription snapshotOnSameTable =
114      new SnapshotDescription(newSnapshotName, TABLE_NAME, SnapshotType.SKIPFLUSH);
115    SnapshotProtos.SnapshotDescription snapshotOnSameTableProto =
116      ProtobufUtil.createHBaseProtosSnapshotDesc(snapshotOnSameTable);
117    Thread second = new Thread("zk-snapshot") {
118      @Override
119      public void run() {
120        try {
121          master.getSnapshotManager().takeSnapshot(snapshotOnSameTableProto);
122        } catch (IOException e) {
123          LOG.error("zk snapshot failed", e);
124          fail("zk snapshot failed");
125        }
126      }
127    };
128    second.start();
129
130    TEST_UTIL.waitFor(2000, () -> sm.isTakingSnapshot(TABLE_NAME));
131    TEST_UTIL.waitFor(60000,
132      () -> sm.isSnapshotDone(snapshotOnSameTableProto) && !sm.isTakingAnySnapshot());
133    SnapshotTestingUtils.confirmSnapshotValid(TEST_UTIL, snapshotProto, TABLE_NAME, CF);
134    SnapshotTestingUtils.confirmSnapshotValid(TEST_UTIL, snapshotOnSameTableProto, TABLE_NAME, CF);
135  }
136
137  @Test
138  public void testItFailsIfTableIsNotDisabledOrEnabled() throws Exception {
139    ProcedureExecutor<MasterProcedureEnv> executor = master.getMasterProcedureExecutor();
140    MasterProcedureEnv env = executor.getEnvironment();
141    TEST_UTIL.getAdmin().disableTable(TABLE_NAME);
142
143    TestEnableTableProcedure enableTable = new TestEnableTableProcedure(
144      master.getMasterProcedureExecutor().getEnvironment(), TABLE_NAME);
145    long enableProcId = executor.submitProcedure(enableTable);
146    TEST_UTIL.waitFor(60000, () -> {
147      Procedure<MasterProcedureEnv> proc = executor.getProcedure(enableProcId);
148      if (proc == null) {
149        return false;
150      }
151      return ((TestEnableTableProcedure) proc).getProcedureState()
152          == ENABLE_TABLE_MARK_REGIONS_ONLINE;
153    });
154
155    // Using a delayed spy ensures we hit the problem state while the table enable procedure
156    // is waiting to run
157    SnapshotProcedure snapshotProc = new SnapshotProcedure(env, snapshotProto);
158    long snapshotProcId = executor.submitProcedure(snapshotProc);
159    TEST_UTIL.waitTableEnabled(TABLE_NAME);
160    // Wait for procedure to run and finish
161    TEST_UTIL.waitFor(60000, () -> executor.getProcedure(snapshotProcId) != null);
162    TEST_UTIL.waitFor(60000, () -> executor.getProcedure(snapshotProcId) == null);
163
164    SnapshotTestingUtils.confirmSnapshotValid(TEST_UTIL, snapshotProto, TABLE_NAME, CF);
165  }
166
167  // Needs to be publicly accessible for Procedure validation
168  public static class TestEnableTableProcedure extends EnableTableProcedure {
169    // Necessary for Procedure validation
170    public TestEnableTableProcedure() {
171    }
172
173    public TestEnableTableProcedure(MasterProcedureEnv env, TableName tableName) {
174      super(env, tableName);
175    }
176
177    public MasterProcedureProtos.EnableTableState getProcedureState() {
178      return getState(stateCount);
179    }
180
181    @Override
182    protected Flow executeFromState(MasterProcedureEnv env,
183      MasterProcedureProtos.EnableTableState state) throws InterruptedException {
184      if (state == ENABLE_TABLE_MARK_REGIONS_ONLINE) {
185        Thread.sleep(10000);
186      }
187
188      return super.executeFromState(env, state);
189    }
190  }
191}