You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@accumulo.apache.org by EdColeman <gi...@git.apache.org> on 2017/01/23 01:46:00 UTC

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

GitHub user EdColeman opened a pull request:

    https://github.com/apache/accumulo/pull/204

    ACCUMULO-4574 Modified TableOperations online to check if table is al\u2026

    ACCUMULO-4574 Modified TableOperations online operation to check if table is already online before executing fate transaction.
    
    When the online command is issued, the fate operation will block if a fate transaction has locked the table. If
    the table is already online, there is no reason to block and then issue the online operation. This modification
    turns the online command into a noop if the table is already online. This change includes an IT
    test that uses a compaction with slow iterator to cause the fate transaction to lock a table and
    then runs the online command - checking that the operation did not block.

You can merge this pull request into a Git repository by running:

    $ git pull https://github.com/EdColeman/accumulo ACCUMULO-4574

Alternatively you can review and apply these changes as the patch at:

    https://github.com/apache/accumulo/pull/204.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

    This closes #204
    
----
commit 058173b59b8e97503a00ec095a9a4235370a9aaa
Author: Ed Coleman <de...@etcoleman.com>
Date:   2017-01-23T01:25:58Z

    ACCUMULO-4574 Modified TableOperations online to check if table is already online before executing fate transaction.
    
    When the online command is issued, the fate operation will block if a fate transaction has locked the table. If
    the table is already online, there is no reason to block and then issue the online operation. This modification
    turns the online command into a noop if the table is already online. This change includes an IT
    test that uses a compaction with slow iterator to cause the fate transaction to lock a table and
    then runs the online command - checking that the operation did not block.

----


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97580943
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    --- End diff --
    
    That's a good point.  The only FATE op I know of that works w/ an offline table is clone.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97467549
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    --- End diff --
    
    The change would be simple enough - but did not know how to test - or if there are any operations that use fate transactions on a table that is already offline.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by joshelser <gi...@git.apache.org>.
Github user joshelser commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97356509
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    --- End diff --
    
    +1


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by joshelser <gi...@git.apache.org>.
Github user joshelser commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97471379
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    --- End diff --
    
    This sounds good to me. This is one of those things that is just not good to try to cover all cases with :). If it's a problem, we can re-visit.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by joshelser <gi...@git.apache.org>.
Github user joshelser commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97355717
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/TableOperationsImpl.java ---
    @@ -1209,6 +1209,15 @@ public void online(String tableName) throws AccumuloSecurityException, AccumuloE
       public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
         checkArgument(tableName != null, "tableName is null");
         String tableId = Tables.getTableId(context.getInstance(), tableName);
    +
    +    /**
    +     * ACCUMULO-4574 if table is already online return without executing fate operation.
    +     */
    +    TableState expectedState = Tables.getTableState(context.getInstance(), tableId);
    --- End diff --
    
    /me lamenting that we don't have a way to do a non-cached read of ZK and are forced to clear the global cache.
    
    If we push this to the Master, would we have more control over reading from ZK? An unnecessary RPC but avoiding the client-side zk cache clear would be well worth it, IMO.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97472003
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    +      lockNodes = new ArrayList<>(lockNodes);
    +      Collections.sort(lockNodes);
    +
    +      for (String node : lockNodes) {
    +
    +        byte[] data = zooCache.get(zPath + "/" + fateId + "/" + node);
    +        String lda[] = new String(data, UTF_8).split(":");
    +
    +        for (String fateString : lda) {
    +
    +          log.trace("Lock: {}:{}: {}", fateId, node, lda);
    +
    +          if (node.contains("prop_debug") && fateString.contains("CompactRange")) {
    +            // found fate associated with table compaction.
    +            log.trace("FOUND - Lock: {}:{}: {}", fateId, node, lda);
    +            return Boolean.TRUE;
    +          }
    +        }
    +      }
    +    }
    +
    +    // did not find appropriate fate transaction for compaction.
    +    return Boolean.FALSE;
    +  }
    +
    +  /**
    +   * Returns the current table state (ONLINE, OFFLINE,...) of named table.
    +   *
    +   * @param connector
    +   *          connector to Accumulo instance
    +   * @param tableName
    +   *          the table name
    +   * @return the current table state
    +   * @throws TableNotFoundException
    +   *           if table does not exist
    +   */
    +  private TableState getTableState(Connector connector, String tableName) throws TableNotFoundException {
    +
    +    String tableId = Tables.getTableId(connector.getInstance(), tableName);
    +
    +    TableState tstate = Tables.getTableState(connector.getInstance(), tableId);
    +
    +    log.trace("tableName: '{}': tableId {}, current state: {}", tableName, tableId, tstate);
    +
    +    return tstate;
    +  }
    +
    +  /**
    +   * Executes TableOperations online command in a separate thread and blocks the current thread until command is complete. The status and time to complete is
    +   * returned in {@code OnlineStatus}
    +   *
    +   * @param tableName
    +   *          the table to set online.
    +   * @return status that can be used to determine if running and timing information when complete.
    +   */
    +
    +  private OnlineStatus onLine(String tableName) {
    +
    +    OnLineThread onLineCmd = new OnLineThread(tableName);
    +    Thread onLineThread = new Thread(onLineCmd);
    +    onLineThread.start();
    +
    +    int count = 0;
    +
    +    while (onLineCmd.getStatus().isRunning()) {
    +
    +      if ((count++ % 10) == 0) {
    +        log.trace("online operation blocked, waiting for it to complete.");
    +      }
    +
    +      try {
    +        Thread.sleep(1000);
    +      } catch (InterruptedException ex) {
    +        // reassert interrupt
    +        Thread.currentThread().interrupt();
    +      }
    +    }
    +
    +    return onLineCmd.getStatus();
    +  }
    +
    +  /**
    +   * Create the provided table and populate with some data using a batch writer. The table is scanned to ensure it was populated as expected.
    +   *
    +   * @param tableName
    +   *          the name of the table
    +   */
    +  private void createData(final String tableName) {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      // create table.
    +      connector.tableOperations().create(tableName);
    +      BatchWriter bw = connector.createBatchWriter(tableName, new BatchWriterConfig());
    +
    +      // populate
    +      for (int i = 0; i < NUM_ROWS; i++) {
    +        Mutation m = new Mutation(new Text(String.format("%05d", i)));
    +        m.put(new Text("col" + Integer.toString((i % 3) + 1)), new Text("qual"), new Value("junk".getBytes(UTF_8)));
    +        bw.addMutation(m);
    +      }
    +      bw.close();
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +      int count = 0;
    +      for (Map.Entry<Key,Value> elt : scanner) {
    +        String expected = String.format("%05d", count);
    +        assert (elt.getKey().getRow().toString().equals(expected));
    +        count++;
    +      }
    +
    +      log.trace("Scan time for {} rows {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +      scanner.close();
    +
    +      if (count != NUM_ROWS) {
    +        throw new IllegalStateException(String.format("Number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +      }
    +    } catch (AccumuloException | AccumuloSecurityException | TableNotFoundException | TableExistsException ex) {
    +      throw new IllegalStateException("Create data failed with exception", ex);
    +    }
    +  }
    +
    +  /**
    +   * Provides timing information and online command status results for online command executed asynchronously. The
    +   */
    +  private static class OnlineStatus {
    +
    +    private long started = 0L;
    +    private long completed = 0L;
    +
    +    // set to true - even before running so that we can block on isRunning before thread may have started.
    +    private final AtomicBoolean isRunning = new AtomicBoolean(Boolean.TRUE);
    +    private Boolean hasError = Boolean.FALSE;
    +    private Exception exception = null;
    +
    +    /**
    +     * Returns true until operation has completed.
    +     *
    +     * @return false when operation has been set complete.
    +     */
    +    Boolean isRunning() {
    +      return isRunning.get();
    +    }
    +
    +    /**
    +     * start timing operation.
    +     */
    +    void setRunning() {
    +      isRunning.set(Boolean.TRUE);
    +      started = System.nanoTime();
    +    }
    +
    +    /**
    +     * stop timing and set completion flag.
    +     */
    +    void setComplete() {
    +      completed = System.nanoTime();
    +      isRunning.set(Boolean.FALSE);
    +    }
    +
    +    /**
    +     * Returns true if setError was called to complete operation.
    +     *
    +     * @return true if an error was set, false otherwise.
    +     */
    +    boolean hasError() {
    +      return hasError;
    +    }
    +
    +    /**
    +     * Returns exception if set by {@code setError}.
    +     *
    +     * @return the exception set with {@code setError} or null if not set.
    +     */
    +    public Exception getException() {
    +      return exception;
    +    }
    +
    +    /**
    +     * Marks the operation as complete, stops timing the operation, with error status and exception that caused the failure.
    +     *
    +     * @param ex
    +     *          the exception that caused failure.
    +     */
    +    public void setError(Exception ex) {
    +      hasError = Boolean.TRUE;
    +      exception = ex;
    +      setComplete();
    +    }
    +
    +    /**
    +     * @return running time in nanoseconds.
    +     */
    +    long runningTime() {
    +      return completed - started;
    +    }
    +  }
    +
    +  /**
    +   * Run online operation in a separate thread. The operation status can be tracked with {@link OnlineStatus} returned by {@link OnLineThread#getStatus}. The
    +   * operation timing is started when run is called. If an exception occurs, it is available in the status.
    +   */
    +  private class OnLineThread implements Runnable {
    +
    +    final String tableName;
    +    final OnlineStatus status;
    +
    +    /**
    +     * Create an instance of this class to set the provided table online.
    +     *
    +     * @param tableName
    +     *          The table name that will be set online.
    +     */
    +    OnLineThread(final String tableName) {
    +      this.tableName = tableName;
    +      this.status = new OnlineStatus();
    +    }
    +
    +    /**
    +     * Update status for timing and execute the online operation.
    +     */
    +    @Override
    +    public void run() {
    +
    +      status.setRunning();
    +
    +      try {
    +
    +        log.trace("Setting {} online", tableName);
    +
    +        getConnector().tableOperations().online(tableName, true);
    +
    +        // stop timing
    +        status.setComplete();
    +        log.trace("Online completed in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +      } catch (Exception ex) {
    +        // set error in status with this exception.
    +        status.setError(ex);
    +      }
    +
    +    }
    +
    +    /**
    +     * Provide OnlineStatus of this operation to determine when complete and timing information
    +     *
    +     * @return OnlineStatus to determine when complete and timing information
    +     */
    +    public OnlineStatus getStatus() {
    +      return status;
    +    }
    +
    +  }
    +
    +  /**
    +   * Instance to create / run a compaction using a slow iterator.
    +   */
    +  private class SlowCompactionRunner implements Runnable {
    +
    +    private final String tableName;
    +
    +    /**
    +     * Create an instance of this class.
    +     *
    +     * @param tableName
    +     *          the name of the table that will be compacted with the slow iterator.
    +     */
    +    SlowCompactionRunner(final String tableName) {
    +      this.tableName = tableName;
    +    }
    +
    +    @Override
    +    public void run() {
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      IteratorSetting slow = new IteratorSetting(30, "slow", SlowIterator.class);
    +      SlowIterator.setSleepTime(slow, 100);
    +
    +      List<IteratorSetting> compactIterators = new ArrayList<>();
    +      compactIterators.add(slow);
    +
    +      log.trace("Slow iterator {}", slow.toString());
    +
    +      try {
    +
    +        Connector connector = getConnector();
    +
    +        log.trace("Start compaction");
    +
    +        connector.tableOperations().compact(tableName, new Text("0"), new Text("z"), compactIterators, true, true);
    +
    +        log.trace("Compaction wait is complete");
    +
    +        log.trace("Slow compaction of {} rows took {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        // validate that number of rows matches expected.
    +
    +        startTimestamp = System.nanoTime();
    +
    +        // validate expected data created and exists in table.
    +
    +        Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +
    +        int count = 0;
    +        for (Map.Entry<Key,Value> elt : scanner) {
    +          String expected = String.format("%05d", count);
    +          assert (elt.getKey().getRow().toString().equals(expected));
    +          count++;
    +        }
    +
    +        log.trace("After compaction, scan time for {} rows {} ms", NUM_ROWS,
    +            TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        if (count != NUM_ROWS) {
    +          throw new IllegalStateException(String.format("After compaction, number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +        }
    +
    +      } catch (TableNotFoundException ex) {
    +        throw new IllegalStateException("test failed, table " + tableName + " does not exist", ex);
    +      } catch (AccumuloSecurityException ex) {
    +        throw new IllegalStateException("test failed, could not add iterator due to security exception", ex);
    +      } catch (AccumuloException ex) {
    +        // test cancels compaction on complete, so ignore it as an exception.
    +        if (!ex.getMessage().contains("Compaction canceled")) {
    --- End diff --
    
    I wrapped it in the IllegalStateException, but the full stack trace is:
    
    Exception in thread "Thread-18" java.lang.IllegalStateException: test failed with an Accumulo exception
    	at org.apache.accumulo.test.functional.TableChangeStateIT$SlowCompactionRunner.run(TableChangeStateIT.java:548)
    	at java.lang.Thread.run(Thread.java:745)
    Caused by: org.apache.accumulo.core.client.AccumuloException: Compaction canceled
    	at org.apache.accumulo.core.client.impl.TableOperationsImpl.doFateOperation(TableOperationsImpl.java:300)
    	at org.apache.accumulo.core.client.impl.TableOperationsImpl.compact(TableOperationsImpl.java:728)
    	at org.apache.accumulo.core.client.impl.TableOperationsImpl.compact(TableOperationsImpl.java:689)
    	at org.apache.accumulo.test.functional.TableChangeStateIT$SlowCompactionRunner.run(TableChangeStateIT.java:513)
    	... 1 more
    Caused by: ThriftTableOperationException(tableId:1, tableName:null, op:COMPACT, type:OTHER, description:Compaction canceled)
    	at org.apache.accumulo.core.master.thrift.FateService$waitForFateOperation_result$waitForFateOperation_resultStandardScheme.read(FateService.java:4243)
    	at org.apache.accumulo.core.master.thrift.FateService$waitForFateOperation_result$waitForFateOperation_resultStandardScheme.read(FateService.java:4212)
    	at org.apache.accumulo.core.master.thrift.FateService$waitForFateOperation_result.read(FateService.java:4146)
    	at org.apache.thrift.TServiceClient.receiveBase(TServiceClient.java:78)
    	at org.apache.accumulo.core.master.thrift.FateService$Client.recv_waitForFateOperation(FateService.java:174)
    	at org.apache.accumulo.core.master.thrift.FateService$Client.waitForFateOperation(FateService.java:159)
    	at org.apache.accumulo.core.client.impl.TableOperationsImpl.waitForFateOperation(TableOperationsImpl.java:233)
    	at org.apache.accumulo.core.client.impl.TableOperationsImpl.doFateOperation(TableOperationsImpl.java:275)
    	... 4 more



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97337920
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    --- End diff --
    
    Do you think we should make the same change for `offline()`?  Make it do nothing if the table is already offline?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by joshelser <gi...@git.apache.org>.
Github user joshelser commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97380760
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    --- End diff --
    
    +1 wrapping it in a simple utility (not in public API) is an easy way forward if the check is necessary/important.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r98584001
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/TableOperationsImpl.java ---
    @@ -1208,11 +1207,20 @@ public void online(String tableName) throws AccumuloSecurityException, AccumuloE
       @Override
       public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
         checkArgument(tableName != null, "tableName is null");
    +
    +    long cacheResetCount = Tables.getCacheResetCount();
    +
         String tableId = Tables.getTableId(context.getInstance(), tableName);
     
         /**
          * ACCUMULO-4574 if table is already online return without executing fate operation.
          */
    +
    +    // getTableId may have side effect of clearing zookeeper cache - no need to clear again.
    +    if (cacheResetCount == Tables.getCacheResetCount()) {
    +      Tables.clearCacheByPath(context.getInstance(), Constants.ZTABLES + "/" + tableId + Constants.ZTABLE_STATE);
    --- End diff --
    
    I have no issue with incorporating this suggestion - leaning towards a getTableState state method with a boolean - however,  a question: Are there other places that would benefit from having the general method? I was assuming that with by passing in a generic path the method would be more general and could be used elsewhere.  Or, are you suggesting that we keep both? I have no strong opinion either way.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo issue #204: ACCUMULO-4574 Modified TableOperations online to check ...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on the issue:

    https://github.com/apache/accumulo/pull/204
  
    Changes have been merged by Keith Turner - closing as no additional work required.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r98457211
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/TableOperationsImpl.java ---
    @@ -1208,11 +1207,20 @@ public void online(String tableName) throws AccumuloSecurityException, AccumuloE
       @Override
       public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
         checkArgument(tableName != null, "tableName is null");
    +
    +    long cacheResetCount = Tables.getCacheResetCount();
    +
         String tableId = Tables.getTableId(context.getInstance(), tableName);
     
         /**
          * ACCUMULO-4574 if table is already online return without executing fate operation.
          */
    +
    +    // getTableId may have side effect of clearing zookeeper cache - no need to clear again.
    +    if (cacheResetCount == Tables.getCacheResetCount()) {
    +      Tables.clearCacheByPath(context.getInstance(), Constants.ZTABLES + "/" + tableId + Constants.ZTABLE_STATE);
    --- End diff --
    
    Might be nice to centralize the zookeeper path construction in Tables class.   Instead of constructing the path here, could have a method called `Tables.clearCachedTableState(String tableId)`.  Alternatively, could add a boolean to  indicates if cache should be used`Tables.getTableState(String tableId, boolean useCache)`.  Structuring the code this way makes it easier to use this functionality elsewhere.  If I wanted to use what you have here elsewhere, I would probably copy and paste the path construction code you have here.  
    
    I think I like adding the boolean to `getTableState` because that makes it easy to change the implementation in the future to not even use zoocache (just go straight to zookeeper) when the boolean is false.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97347490
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    +      lockNodes = new ArrayList<>(lockNodes);
    +      Collections.sort(lockNodes);
    +
    +      for (String node : lockNodes) {
    +
    +        byte[] data = zooCache.get(zPath + "/" + fateId + "/" + node);
    +        String lda[] = new String(data, UTF_8).split(":");
    +
    +        for (String fateString : lda) {
    +
    +          log.trace("Lock: {}:{}: {}", fateId, node, lda);
    +
    +          if (node.contains("prop_debug") && fateString.contains("CompactRange")) {
    +            // found fate associated with table compaction.
    +            log.trace("FOUND - Lock: {}:{}: {}", fateId, node, lda);
    +            return Boolean.TRUE;
    +          }
    +        }
    +      }
    +    }
    +
    +    // did not find appropriate fate transaction for compaction.
    +    return Boolean.FALSE;
    +  }
    +
    +  /**
    +   * Returns the current table state (ONLINE, OFFLINE,...) of named table.
    +   *
    +   * @param connector
    +   *          connector to Accumulo instance
    +   * @param tableName
    +   *          the table name
    +   * @return the current table state
    +   * @throws TableNotFoundException
    +   *           if table does not exist
    +   */
    +  private TableState getTableState(Connector connector, String tableName) throws TableNotFoundException {
    +
    +    String tableId = Tables.getTableId(connector.getInstance(), tableName);
    +
    +    TableState tstate = Tables.getTableState(connector.getInstance(), tableId);
    +
    +    log.trace("tableName: '{}': tableId {}, current state: {}", tableName, tableId, tstate);
    +
    +    return tstate;
    +  }
    +
    +  /**
    +   * Executes TableOperations online command in a separate thread and blocks the current thread until command is complete. The status and time to complete is
    +   * returned in {@code OnlineStatus}
    +   *
    +   * @param tableName
    +   *          the table to set online.
    +   * @return status that can be used to determine if running and timing information when complete.
    +   */
    +
    +  private OnlineStatus onLine(String tableName) {
    +
    +    OnLineThread onLineCmd = new OnLineThread(tableName);
    +    Thread onLineThread = new Thread(onLineCmd);
    +    onLineThread.start();
    +
    +    int count = 0;
    +
    +    while (onLineCmd.getStatus().isRunning()) {
    +
    +      if ((count++ % 10) == 0) {
    +        log.trace("online operation blocked, waiting for it to complete.");
    +      }
    +
    +      try {
    +        Thread.sleep(1000);
    +      } catch (InterruptedException ex) {
    +        // reassert interrupt
    +        Thread.currentThread().interrupt();
    +      }
    +    }
    +
    +    return onLineCmd.getStatus();
    +  }
    +
    +  /**
    +   * Create the provided table and populate with some data using a batch writer. The table is scanned to ensure it was populated as expected.
    +   *
    +   * @param tableName
    +   *          the name of the table
    +   */
    +  private void createData(final String tableName) {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      // create table.
    +      connector.tableOperations().create(tableName);
    +      BatchWriter bw = connector.createBatchWriter(tableName, new BatchWriterConfig());
    +
    +      // populate
    +      for (int i = 0; i < NUM_ROWS; i++) {
    +        Mutation m = new Mutation(new Text(String.format("%05d", i)));
    +        m.put(new Text("col" + Integer.toString((i % 3) + 1)), new Text("qual"), new Value("junk".getBytes(UTF_8)));
    +        bw.addMutation(m);
    +      }
    +      bw.close();
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +      int count = 0;
    +      for (Map.Entry<Key,Value> elt : scanner) {
    +        String expected = String.format("%05d", count);
    +        assert (elt.getKey().getRow().toString().equals(expected));
    +        count++;
    +      }
    +
    +      log.trace("Scan time for {} rows {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +      scanner.close();
    +
    +      if (count != NUM_ROWS) {
    +        throw new IllegalStateException(String.format("Number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +      }
    +    } catch (AccumuloException | AccumuloSecurityException | TableNotFoundException | TableExistsException ex) {
    +      throw new IllegalStateException("Create data failed with exception", ex);
    +    }
    +  }
    +
    +  /**
    +   * Provides timing information and online command status results for online command executed asynchronously. The
    +   */
    +  private static class OnlineStatus {
    +
    +    private long started = 0L;
    +    private long completed = 0L;
    +
    +    // set to true - even before running so that we can block on isRunning before thread may have started.
    +    private final AtomicBoolean isRunning = new AtomicBoolean(Boolean.TRUE);
    +    private Boolean hasError = Boolean.FALSE;
    +    private Exception exception = null;
    +
    +    /**
    +     * Returns true until operation has completed.
    +     *
    +     * @return false when operation has been set complete.
    +     */
    +    Boolean isRunning() {
    +      return isRunning.get();
    +    }
    +
    +    /**
    +     * start timing operation.
    +     */
    +    void setRunning() {
    +      isRunning.set(Boolean.TRUE);
    +      started = System.nanoTime();
    +    }
    +
    +    /**
    +     * stop timing and set completion flag.
    +     */
    +    void setComplete() {
    +      completed = System.nanoTime();
    +      isRunning.set(Boolean.FALSE);
    +    }
    +
    +    /**
    +     * Returns true if setError was called to complete operation.
    +     *
    +     * @return true if an error was set, false otherwise.
    +     */
    +    boolean hasError() {
    +      return hasError;
    +    }
    +
    +    /**
    +     * Returns exception if set by {@code setError}.
    +     *
    +     * @return the exception set with {@code setError} or null if not set.
    +     */
    +    public Exception getException() {
    +      return exception;
    +    }
    +
    +    /**
    +     * Marks the operation as complete, stops timing the operation, with error status and exception that caused the failure.
    +     *
    +     * @param ex
    +     *          the exception that caused failure.
    +     */
    +    public void setError(Exception ex) {
    +      hasError = Boolean.TRUE;
    +      exception = ex;
    +      setComplete();
    +    }
    +
    +    /**
    +     * @return running time in nanoseconds.
    +     */
    +    long runningTime() {
    +      return completed - started;
    +    }
    +  }
    +
    +  /**
    +   * Run online operation in a separate thread. The operation status can be tracked with {@link OnlineStatus} returned by {@link OnLineThread#getStatus}. The
    +   * operation timing is started when run is called. If an exception occurs, it is available in the status.
    +   */
    +  private class OnLineThread implements Runnable {
    --- End diff --
    
    Using FutureTask may simplify this class... or it may not.  I am not sure, its just something I thought of while looking at this code.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by phrocker <gi...@git.apache.org>.
Github user phrocker commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97362683
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/TableOperationsImpl.java ---
    @@ -1209,6 +1209,15 @@ public void online(String tableName) throws AccumuloSecurityException, AccumuloE
       public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
         checkArgument(tableName != null, "tableName is null");
         String tableId = Tables.getTableId(context.getInstance(), tableName);
    +
    +    /**
    +     * ACCUMULO-4574 if table is already online return without executing fate operation.
    +     */
    +    TableState expectedState = Tables.getTableState(context.getInstance(), tableId);
    --- End diff --
    
    Why not clear the zoocache then? Further, why not clear that individual path to avoid impacting every other thread?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by joshelser <gi...@git.apache.org>.
Github user joshelser commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97471864
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    +      lockNodes = new ArrayList<>(lockNodes);
    +      Collections.sort(lockNodes);
    +
    +      for (String node : lockNodes) {
    +
    +        byte[] data = zooCache.get(zPath + "/" + fateId + "/" + node);
    +        String lda[] = new String(data, UTF_8).split(":");
    +
    +        for (String fateString : lda) {
    +
    +          log.trace("Lock: {}:{}: {}", fateId, node, lda);
    +
    +          if (node.contains("prop_debug") && fateString.contains("CompactRange")) {
    +            // found fate associated with table compaction.
    +            log.trace("FOUND - Lock: {}:{}: {}", fateId, node, lda);
    +            return Boolean.TRUE;
    +          }
    +        }
    +      }
    +    }
    +
    +    // did not find appropriate fate transaction for compaction.
    +    return Boolean.FALSE;
    +  }
    +
    +  /**
    +   * Returns the current table state (ONLINE, OFFLINE,...) of named table.
    +   *
    +   * @param connector
    +   *          connector to Accumulo instance
    +   * @param tableName
    +   *          the table name
    +   * @return the current table state
    +   * @throws TableNotFoundException
    +   *           if table does not exist
    +   */
    +  private TableState getTableState(Connector connector, String tableName) throws TableNotFoundException {
    +
    +    String tableId = Tables.getTableId(connector.getInstance(), tableName);
    +
    +    TableState tstate = Tables.getTableState(connector.getInstance(), tableId);
    +
    +    log.trace("tableName: '{}': tableId {}, current state: {}", tableName, tableId, tstate);
    +
    +    return tstate;
    +  }
    +
    +  /**
    +   * Executes TableOperations online command in a separate thread and blocks the current thread until command is complete. The status and time to complete is
    +   * returned in {@code OnlineStatus}
    +   *
    +   * @param tableName
    +   *          the table to set online.
    +   * @return status that can be used to determine if running and timing information when complete.
    +   */
    +
    +  private OnlineStatus onLine(String tableName) {
    +
    +    OnLineThread onLineCmd = new OnLineThread(tableName);
    +    Thread onLineThread = new Thread(onLineCmd);
    +    onLineThread.start();
    +
    +    int count = 0;
    +
    +    while (onLineCmd.getStatus().isRunning()) {
    +
    +      if ((count++ % 10) == 0) {
    +        log.trace("online operation blocked, waiting for it to complete.");
    +      }
    +
    +      try {
    +        Thread.sleep(1000);
    +      } catch (InterruptedException ex) {
    +        // reassert interrupt
    +        Thread.currentThread().interrupt();
    +      }
    +    }
    +
    +    return onLineCmd.getStatus();
    +  }
    +
    +  /**
    +   * Create the provided table and populate with some data using a batch writer. The table is scanned to ensure it was populated as expected.
    +   *
    +   * @param tableName
    +   *          the name of the table
    +   */
    +  private void createData(final String tableName) {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      // create table.
    +      connector.tableOperations().create(tableName);
    +      BatchWriter bw = connector.createBatchWriter(tableName, new BatchWriterConfig());
    +
    +      // populate
    +      for (int i = 0; i < NUM_ROWS; i++) {
    +        Mutation m = new Mutation(new Text(String.format("%05d", i)));
    +        m.put(new Text("col" + Integer.toString((i % 3) + 1)), new Text("qual"), new Value("junk".getBytes(UTF_8)));
    +        bw.addMutation(m);
    +      }
    +      bw.close();
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +      int count = 0;
    +      for (Map.Entry<Key,Value> elt : scanner) {
    +        String expected = String.format("%05d", count);
    +        assert (elt.getKey().getRow().toString().equals(expected));
    +        count++;
    +      }
    +
    +      log.trace("Scan time for {} rows {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +      scanner.close();
    +
    +      if (count != NUM_ROWS) {
    +        throw new IllegalStateException(String.format("Number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +      }
    +    } catch (AccumuloException | AccumuloSecurityException | TableNotFoundException | TableExistsException ex) {
    +      throw new IllegalStateException("Create data failed with exception", ex);
    +    }
    +  }
    +
    +  /**
    +   * Provides timing information and online command status results for online command executed asynchronously. The
    +   */
    +  private static class OnlineStatus {
    +
    +    private long started = 0L;
    +    private long completed = 0L;
    +
    +    // set to true - even before running so that we can block on isRunning before thread may have started.
    +    private final AtomicBoolean isRunning = new AtomicBoolean(Boolean.TRUE);
    +    private Boolean hasError = Boolean.FALSE;
    +    private Exception exception = null;
    +
    +    /**
    +     * Returns true until operation has completed.
    +     *
    +     * @return false when operation has been set complete.
    +     */
    +    Boolean isRunning() {
    +      return isRunning.get();
    +    }
    +
    +    /**
    +     * start timing operation.
    +     */
    +    void setRunning() {
    +      isRunning.set(Boolean.TRUE);
    +      started = System.nanoTime();
    +    }
    +
    +    /**
    +     * stop timing and set completion flag.
    +     */
    +    void setComplete() {
    +      completed = System.nanoTime();
    +      isRunning.set(Boolean.FALSE);
    +    }
    +
    +    /**
    +     * Returns true if setError was called to complete operation.
    +     *
    +     * @return true if an error was set, false otherwise.
    +     */
    +    boolean hasError() {
    +      return hasError;
    +    }
    +
    +    /**
    +     * Returns exception if set by {@code setError}.
    +     *
    +     * @return the exception set with {@code setError} or null if not set.
    +     */
    +    public Exception getException() {
    +      return exception;
    +    }
    +
    +    /**
    +     * Marks the operation as complete, stops timing the operation, with error status and exception that caused the failure.
    +     *
    +     * @param ex
    +     *          the exception that caused failure.
    +     */
    +    public void setError(Exception ex) {
    +      hasError = Boolean.TRUE;
    +      exception = ex;
    +      setComplete();
    +    }
    +
    +    /**
    +     * @return running time in nanoseconds.
    +     */
    +    long runningTime() {
    +      return completed - started;
    +    }
    +  }
    +
    +  /**
    +   * Run online operation in a separate thread. The operation status can be tracked with {@link OnlineStatus} returned by {@link OnLineThread#getStatus}. The
    +   * operation timing is started when run is called. If an exception occurs, it is available in the status.
    +   */
    +  private class OnLineThread implements Runnable {
    +
    +    final String tableName;
    +    final OnlineStatus status;
    +
    +    /**
    +     * Create an instance of this class to set the provided table online.
    +     *
    +     * @param tableName
    +     *          The table name that will be set online.
    +     */
    +    OnLineThread(final String tableName) {
    +      this.tableName = tableName;
    +      this.status = new OnlineStatus();
    +    }
    +
    +    /**
    +     * Update status for timing and execute the online operation.
    +     */
    +    @Override
    +    public void run() {
    +
    +      status.setRunning();
    +
    +      try {
    +
    +        log.trace("Setting {} online", tableName);
    +
    +        getConnector().tableOperations().online(tableName, true);
    +
    +        // stop timing
    +        status.setComplete();
    +        log.trace("Online completed in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +      } catch (Exception ex) {
    +        // set error in status with this exception.
    +        status.setError(ex);
    +      }
    +
    +    }
    +
    +    /**
    +     * Provide OnlineStatus of this operation to determine when complete and timing information
    +     *
    +     * @return OnlineStatus to determine when complete and timing information
    +     */
    +    public OnlineStatus getStatus() {
    +      return status;
    +    }
    +
    +  }
    +
    +  /**
    +   * Instance to create / run a compaction using a slow iterator.
    +   */
    +  private class SlowCompactionRunner implements Runnable {
    +
    +    private final String tableName;
    +
    +    /**
    +     * Create an instance of this class.
    +     *
    +     * @param tableName
    +     *          the name of the table that will be compacted with the slow iterator.
    +     */
    +    SlowCompactionRunner(final String tableName) {
    +      this.tableName = tableName;
    +    }
    +
    +    @Override
    +    public void run() {
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      IteratorSetting slow = new IteratorSetting(30, "slow", SlowIterator.class);
    +      SlowIterator.setSleepTime(slow, 100);
    +
    +      List<IteratorSetting> compactIterators = new ArrayList<>();
    +      compactIterators.add(slow);
    +
    +      log.trace("Slow iterator {}", slow.toString());
    +
    +      try {
    +
    +        Connector connector = getConnector();
    +
    +        log.trace("Start compaction");
    +
    +        connector.tableOperations().compact(tableName, new Text("0"), new Text("z"), compactIterators, true, true);
    +
    +        log.trace("Compaction wait is complete");
    +
    +        log.trace("Slow compaction of {} rows took {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        // validate that number of rows matches expected.
    +
    +        startTimestamp = System.nanoTime();
    +
    +        // validate expected data created and exists in table.
    +
    +        Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +
    +        int count = 0;
    +        for (Map.Entry<Key,Value> elt : scanner) {
    +          String expected = String.format("%05d", count);
    +          assert (elt.getKey().getRow().toString().equals(expected));
    +          count++;
    +        }
    +
    +        log.trace("After compaction, scan time for {} rows {} ms", NUM_ROWS,
    +            TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        if (count != NUM_ROWS) {
    +          throw new IllegalStateException(String.format("After compaction, number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +        }
    +
    +      } catch (TableNotFoundException ex) {
    +        throw new IllegalStateException("test failed, table " + tableName + " does not exist", ex);
    +      } catch (AccumuloSecurityException ex) {
    +        throw new IllegalStateException("test failed, could not add iterator due to security exception", ex);
    +      } catch (AccumuloException ex) {
    +        // test cancels compaction on complete, so ignore it as an exception.
    +        if (!ex.getMessage().contains("Compaction canceled")) {
    --- End diff --
    
    Great, sounds good. Was mostly curious how it was actually propagated :)


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97465600
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    --- End diff --
    
    This was to print the error stack trace to the log - the assertFalse immediately following will fail the test,


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by phrocker <gi...@git.apache.org>.
Github user phrocker commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97363650
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    --- End diff --
    
    This is kind of fate specific, I agree,  and could be useful in a fate utility class. 


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo issue #204: ACCUMULO-4574 Modified TableOperations online to check ...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on the issue:

    https://github.com/apache/accumulo/pull/204
  
    I merged these changes 15ae69c.  I reorganized imports for TableOperationsImpl in the squashed commit.  In the merged to 1.8 I moved the new IT and changed the class it extended.  I ran the new IT on each branch (1.7, 1.8, master) and I also ran `mvn clean verify -DskipTests` on each branch.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97347789
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    --- End diff --
    
    This code assumes fate internals work a certain way.  If fate internals a changed, then someone would need to know to change this method OR we need to be 100% sure this will fail if internals are changed.  I was wondering if internal FATE APIs could be used here.  I looked at ZooStore and I am not sure if has everything you need.   I am wondering if the test would be ok w/o this method?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman closed the pull request at:

    https://github.com/apache/accumulo/pull/204


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r98455152
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/Tables.java ---
    @@ -149,6 +149,28 @@ public static void clearCache(Instance instance) {
         getZooCache(instance).clear(ZooUtil.getRoot(instance) + Constants.ZNAMESPACES);
       }
     
    +  /**
    +   * Clears the zoo cache from instance/root/{PATH}
    +   *
    +   * @param instance
    +   *          The Accumulo Instance
    +   * @param zooPath
    +   *          A zookeeper path
    +   */
    +  public static void clearCacheByPath(Instance instance, final String zooPath) {
    +
    +    String thePath;
    +
    +    if (zooPath.startsWith("/")) {
    +      thePath = zooPath;
    +    } else {
    +      thePath = "/" + zooPath;
    +    }
    +
    +    getZooCache(instance).clear(ZooUtil.getRoot(instance) + thePath);
    --- End diff --
    
    Do you know if this zoocache method handles path normalization?  For example will is treat `/a/b/c/` and `/a//b/c/` the same?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by joshelser <gi...@git.apache.org>.
Github user joshelser commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97356458
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    --- End diff --
    
    If you can, please extend AccumuloClusterIT instead of ConfigurableMacIT.
    
    In most cases, there's no reason to require the use of a minicluster. Extending AccumuloClusterIT provides the ability to also run against a real Accumulo cluster (reducing the time it takes to run tests).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97346413
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    --- End diff --
    
    Will Java GC or swap cause this to fail when there is no problem?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97467208
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    --- End diff --
    
    Increased from 2 seconds to be < compaction time which is 100 seconds and can be modified by changing the number of data rows or scanner sleep time, (currently 1000 rows and 100 ms sleep.)  On my VM this test is taking ~5 ms.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97365531
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/TableOperationsImpl.java ---
    @@ -1209,6 +1209,15 @@ public void online(String tableName) throws AccumuloSecurityException, AccumuloE
       public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
         checkArgument(tableName != null, "tableName is null");
         String tableId = Tables.getTableId(context.getInstance(), tableName);
    +
    +    /**
    +     * ACCUMULO-4574 if table is already online return without executing fate operation.
    +     */
    +    TableState expectedState = Tables.getTableState(context.getInstance(), tableId);
    --- End diff --
    
    ZooCache has a method `clear(String zPath)` that could be called by `Tables` to clear specific info.   Could open a follow on ticket to look into using this.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r98583191
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/Tables.java ---
    @@ -149,6 +149,28 @@ public static void clearCache(Instance instance) {
         getZooCache(instance).clear(ZooUtil.getRoot(instance) + Constants.ZNAMESPACES);
       }
     
    +  /**
    +   * Clears the zoo cache from instance/root/{PATH}
    +   *
    +   * @param instance
    +   *          The Accumulo Instance
    +   * @param zooPath
    +   *          A zookeeper path
    +   */
    +  public static void clearCacheByPath(Instance instance, final String zooPath) {
    +
    +    String thePath;
    +
    +    if (zooPath.startsWith("/")) {
    +      thePath = zooPath;
    +    } else {
    +      thePath = "/" + zooPath;
    +    }
    +
    +    getZooCache(instance).clear(ZooUtil.getRoot(instance) + thePath);
    --- End diff --
    
    I was not sure - so I was only adding the slash if it was not present.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97339379
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    --- End diff --
    
    why not fail the test here?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97337611
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/TableOperationsImpl.java ---
    @@ -1209,6 +1209,15 @@ public void online(String tableName) throws AccumuloSecurityException, AccumuloE
       public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
         checkArgument(tableName != null, "tableName is null");
         String tableId = Tables.getTableId(context.getInstance(), tableName);
    +
    +    /**
    +     * ACCUMULO-4574 if table is already online return without executing fate operation.
    +     */
    +    TableState expectedState = Tables.getTableState(context.getInstance(), tableId);
    --- End diff --
    
    The implementation of this uses zoo cache.   So it may return stale information.  Before getting the table state, could call `Tables.clearCache()`.  This will force the table state to be read from zookeeper.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r98688065
  
    --- Diff: core/src/main/java/org/apache/accumulo/core/client/impl/TableOperationsImpl.java ---
    @@ -1208,11 +1207,20 @@ public void online(String tableName) throws AccumuloSecurityException, AccumuloE
       @Override
       public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException {
         checkArgument(tableName != null, "tableName is null");
    +
    +    long cacheResetCount = Tables.getCacheResetCount();
    +
         String tableId = Tables.getTableId(context.getInstance(), tableName);
     
         /**
          * ACCUMULO-4574 if table is already online return without executing fate operation.
          */
    +
    +    // getTableId may have side effect of clearing zookeeper cache - no need to clear again.
    +    if (cacheResetCount == Tables.getCacheResetCount()) {
    +      Tables.clearCacheByPath(context.getInstance(), Constants.ZTABLES + "/" + tableId + Constants.ZTABLE_STATE);
    --- End diff --
    
    I suspect other methods in Tables class would benefit from having a boolean that indicates if cache should be used.  If you go with the boolean option, could possibly have a private method ` Tables.clearCachedTableState(String tableId)` that's used in the implementation.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by keith-turner <gi...@git.apache.org>.
Github user keith-turner commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97345685
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    +      lockNodes = new ArrayList<>(lockNodes);
    +      Collections.sort(lockNodes);
    +
    +      for (String node : lockNodes) {
    +
    +        byte[] data = zooCache.get(zPath + "/" + fateId + "/" + node);
    +        String lda[] = new String(data, UTF_8).split(":");
    +
    +        for (String fateString : lda) {
    +
    +          log.trace("Lock: {}:{}: {}", fateId, node, lda);
    +
    +          if (node.contains("prop_debug") && fateString.contains("CompactRange")) {
    +            // found fate associated with table compaction.
    +            log.trace("FOUND - Lock: {}:{}: {}", fateId, node, lda);
    +            return Boolean.TRUE;
    +          }
    +        }
    +      }
    +    }
    +
    +    // did not find appropriate fate transaction for compaction.
    +    return Boolean.FALSE;
    +  }
    +
    +  /**
    +   * Returns the current table state (ONLINE, OFFLINE,...) of named table.
    +   *
    +   * @param connector
    +   *          connector to Accumulo instance
    +   * @param tableName
    +   *          the table name
    +   * @return the current table state
    +   * @throws TableNotFoundException
    +   *           if table does not exist
    +   */
    +  private TableState getTableState(Connector connector, String tableName) throws TableNotFoundException {
    +
    +    String tableId = Tables.getTableId(connector.getInstance(), tableName);
    +
    +    TableState tstate = Tables.getTableState(connector.getInstance(), tableId);
    +
    +    log.trace("tableName: '{}': tableId {}, current state: {}", tableName, tableId, tstate);
    +
    +    return tstate;
    +  }
    +
    +  /**
    +   * Executes TableOperations online command in a separate thread and blocks the current thread until command is complete. The status and time to complete is
    +   * returned in {@code OnlineStatus}
    +   *
    +   * @param tableName
    +   *          the table to set online.
    +   * @return status that can be used to determine if running and timing information when complete.
    +   */
    +
    +  private OnlineStatus onLine(String tableName) {
    +
    +    OnLineThread onLineCmd = new OnLineThread(tableName);
    +    Thread onLineThread = new Thread(onLineCmd);
    +    onLineThread.start();
    +
    +    int count = 0;
    +
    +    while (onLineCmd.getStatus().isRunning()) {
    +
    +      if ((count++ % 10) == 0) {
    +        log.trace("online operation blocked, waiting for it to complete.");
    +      }
    +
    +      try {
    +        Thread.sleep(1000);
    +      } catch (InterruptedException ex) {
    +        // reassert interrupt
    +        Thread.currentThread().interrupt();
    +      }
    +    }
    +
    +    return onLineCmd.getStatus();
    +  }
    +
    +  /**
    +   * Create the provided table and populate with some data using a batch writer. The table is scanned to ensure it was populated as expected.
    +   *
    +   * @param tableName
    +   *          the name of the table
    +   */
    +  private void createData(final String tableName) {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      // create table.
    +      connector.tableOperations().create(tableName);
    +      BatchWriter bw = connector.createBatchWriter(tableName, new BatchWriterConfig());
    +
    +      // populate
    +      for (int i = 0; i < NUM_ROWS; i++) {
    +        Mutation m = new Mutation(new Text(String.format("%05d", i)));
    +        m.put(new Text("col" + Integer.toString((i % 3) + 1)), new Text("qual"), new Value("junk".getBytes(UTF_8)));
    +        bw.addMutation(m);
    +      }
    +      bw.close();
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +      int count = 0;
    +      for (Map.Entry<Key,Value> elt : scanner) {
    +        String expected = String.format("%05d", count);
    +        assert (elt.getKey().getRow().toString().equals(expected));
    +        count++;
    +      }
    +
    +      log.trace("Scan time for {} rows {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +      scanner.close();
    +
    +      if (count != NUM_ROWS) {
    +        throw new IllegalStateException(String.format("Number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +      }
    +    } catch (AccumuloException | AccumuloSecurityException | TableNotFoundException | TableExistsException ex) {
    +      throw new IllegalStateException("Create data failed with exception", ex);
    +    }
    +  }
    +
    +  /**
    +   * Provides timing information and online command status results for online command executed asynchronously. The
    +   */
    +  private static class OnlineStatus {
    --- End diff --
    
    Since multiple instance vars are being written and read by multiple threads, should probably synchronize all methods in this class.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by EdColeman <gi...@git.apache.org>.
Github user EdColeman commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97466408
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    +      lockNodes = new ArrayList<>(lockNodes);
    +      Collections.sort(lockNodes);
    +
    +      for (String node : lockNodes) {
    +
    +        byte[] data = zooCache.get(zPath + "/" + fateId + "/" + node);
    +        String lda[] = new String(data, UTF_8).split(":");
    +
    +        for (String fateString : lda) {
    +
    +          log.trace("Lock: {}:{}: {}", fateId, node, lda);
    +
    +          if (node.contains("prop_debug") && fateString.contains("CompactRange")) {
    +            // found fate associated with table compaction.
    +            log.trace("FOUND - Lock: {}:{}: {}", fateId, node, lda);
    +            return Boolean.TRUE;
    +          }
    +        }
    +      }
    +    }
    +
    +    // did not find appropriate fate transaction for compaction.
    +    return Boolean.FALSE;
    +  }
    +
    +  /**
    +   * Returns the current table state (ONLINE, OFFLINE,...) of named table.
    +   *
    +   * @param connector
    +   *          connector to Accumulo instance
    +   * @param tableName
    +   *          the table name
    +   * @return the current table state
    +   * @throws TableNotFoundException
    +   *           if table does not exist
    +   */
    +  private TableState getTableState(Connector connector, String tableName) throws TableNotFoundException {
    +
    +    String tableId = Tables.getTableId(connector.getInstance(), tableName);
    +
    +    TableState tstate = Tables.getTableState(connector.getInstance(), tableId);
    +
    +    log.trace("tableName: '{}': tableId {}, current state: {}", tableName, tableId, tstate);
    +
    +    return tstate;
    +  }
    +
    +  /**
    +   * Executes TableOperations online command in a separate thread and blocks the current thread until command is complete. The status and time to complete is
    +   * returned in {@code OnlineStatus}
    +   *
    +   * @param tableName
    +   *          the table to set online.
    +   * @return status that can be used to determine if running and timing information when complete.
    +   */
    +
    +  private OnlineStatus onLine(String tableName) {
    +
    +    OnLineThread onLineCmd = new OnLineThread(tableName);
    +    Thread onLineThread = new Thread(onLineCmd);
    +    onLineThread.start();
    +
    +    int count = 0;
    +
    +    while (onLineCmd.getStatus().isRunning()) {
    +
    +      if ((count++ % 10) == 0) {
    +        log.trace("online operation blocked, waiting for it to complete.");
    +      }
    +
    +      try {
    +        Thread.sleep(1000);
    +      } catch (InterruptedException ex) {
    +        // reassert interrupt
    +        Thread.currentThread().interrupt();
    +      }
    +    }
    +
    +    return onLineCmd.getStatus();
    +  }
    +
    +  /**
    +   * Create the provided table and populate with some data using a batch writer. The table is scanned to ensure it was populated as expected.
    +   *
    +   * @param tableName
    +   *          the name of the table
    +   */
    +  private void createData(final String tableName) {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      // create table.
    +      connector.tableOperations().create(tableName);
    +      BatchWriter bw = connector.createBatchWriter(tableName, new BatchWriterConfig());
    +
    +      // populate
    +      for (int i = 0; i < NUM_ROWS; i++) {
    +        Mutation m = new Mutation(new Text(String.format("%05d", i)));
    +        m.put(new Text("col" + Integer.toString((i % 3) + 1)), new Text("qual"), new Value("junk".getBytes(UTF_8)));
    +        bw.addMutation(m);
    +      }
    +      bw.close();
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +      int count = 0;
    +      for (Map.Entry<Key,Value> elt : scanner) {
    +        String expected = String.format("%05d", count);
    +        assert (elt.getKey().getRow().toString().equals(expected));
    +        count++;
    +      }
    +
    +      log.trace("Scan time for {} rows {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +      scanner.close();
    +
    +      if (count != NUM_ROWS) {
    +        throw new IllegalStateException(String.format("Number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +      }
    +    } catch (AccumuloException | AccumuloSecurityException | TableNotFoundException | TableExistsException ex) {
    +      throw new IllegalStateException("Create data failed with exception", ex);
    +    }
    +  }
    +
    +  /**
    +   * Provides timing information and online command status results for online command executed asynchronously. The
    +   */
    +  private static class OnlineStatus {
    +
    +    private long started = 0L;
    +    private long completed = 0L;
    +
    +    // set to true - even before running so that we can block on isRunning before thread may have started.
    +    private final AtomicBoolean isRunning = new AtomicBoolean(Boolean.TRUE);
    +    private Boolean hasError = Boolean.FALSE;
    +    private Exception exception = null;
    +
    +    /**
    +     * Returns true until operation has completed.
    +     *
    +     * @return false when operation has been set complete.
    +     */
    +    Boolean isRunning() {
    +      return isRunning.get();
    +    }
    +
    +    /**
    +     * start timing operation.
    +     */
    +    void setRunning() {
    +      isRunning.set(Boolean.TRUE);
    +      started = System.nanoTime();
    +    }
    +
    +    /**
    +     * stop timing and set completion flag.
    +     */
    +    void setComplete() {
    +      completed = System.nanoTime();
    +      isRunning.set(Boolean.FALSE);
    +    }
    +
    +    /**
    +     * Returns true if setError was called to complete operation.
    +     *
    +     * @return true if an error was set, false otherwise.
    +     */
    +    boolean hasError() {
    +      return hasError;
    +    }
    +
    +    /**
    +     * Returns exception if set by {@code setError}.
    +     *
    +     * @return the exception set with {@code setError} or null if not set.
    +     */
    +    public Exception getException() {
    +      return exception;
    +    }
    +
    +    /**
    +     * Marks the operation as complete, stops timing the operation, with error status and exception that caused the failure.
    +     *
    +     * @param ex
    +     *          the exception that caused failure.
    +     */
    +    public void setError(Exception ex) {
    +      hasError = Boolean.TRUE;
    +      exception = ex;
    +      setComplete();
    +    }
    +
    +    /**
    +     * @return running time in nanoseconds.
    +     */
    +    long runningTime() {
    +      return completed - started;
    +    }
    +  }
    +
    +  /**
    +   * Run online operation in a separate thread. The operation status can be tracked with {@link OnlineStatus} returned by {@link OnLineThread#getStatus}. The
    +   * operation timing is started when run is called. If an exception occurs, it is available in the status.
    +   */
    +  private class OnLineThread implements Runnable {
    +
    +    final String tableName;
    +    final OnlineStatus status;
    +
    +    /**
    +     * Create an instance of this class to set the provided table online.
    +     *
    +     * @param tableName
    +     *          The table name that will be set online.
    +     */
    +    OnLineThread(final String tableName) {
    +      this.tableName = tableName;
    +      this.status = new OnlineStatus();
    +    }
    +
    +    /**
    +     * Update status for timing and execute the online operation.
    +     */
    +    @Override
    +    public void run() {
    +
    +      status.setRunning();
    +
    +      try {
    +
    +        log.trace("Setting {} online", tableName);
    +
    +        getConnector().tableOperations().online(tableName, true);
    +
    +        // stop timing
    +        status.setComplete();
    +        log.trace("Online completed in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +      } catch (Exception ex) {
    +        // set error in status with this exception.
    +        status.setError(ex);
    +      }
    +
    +    }
    +
    +    /**
    +     * Provide OnlineStatus of this operation to determine when complete and timing information
    +     *
    +     * @return OnlineStatus to determine when complete and timing information
    +     */
    +    public OnlineStatus getStatus() {
    +      return status;
    +    }
    +
    +  }
    +
    +  /**
    +   * Instance to create / run a compaction using a slow iterator.
    +   */
    +  private class SlowCompactionRunner implements Runnable {
    +
    +    private final String tableName;
    +
    +    /**
    +     * Create an instance of this class.
    +     *
    +     * @param tableName
    +     *          the name of the table that will be compacted with the slow iterator.
    +     */
    +    SlowCompactionRunner(final String tableName) {
    +      this.tableName = tableName;
    +    }
    +
    +    @Override
    +    public void run() {
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      IteratorSetting slow = new IteratorSetting(30, "slow", SlowIterator.class);
    +      SlowIterator.setSleepTime(slow, 100);
    +
    +      List<IteratorSetting> compactIterators = new ArrayList<>();
    +      compactIterators.add(slow);
    +
    +      log.trace("Slow iterator {}", slow.toString());
    +
    +      try {
    +
    +        Connector connector = getConnector();
    +
    +        log.trace("Start compaction");
    +
    +        connector.tableOperations().compact(tableName, new Text("0"), new Text("z"), compactIterators, true, true);
    +
    +        log.trace("Compaction wait is complete");
    +
    +        log.trace("Slow compaction of {} rows took {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        // validate that number of rows matches expected.
    +
    +        startTimestamp = System.nanoTime();
    +
    +        // validate expected data created and exists in table.
    +
    +        Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +
    +        int count = 0;
    +        for (Map.Entry<Key,Value> elt : scanner) {
    +          String expected = String.format("%05d", count);
    +          assert (elt.getKey().getRow().toString().equals(expected));
    +          count++;
    +        }
    +
    +        log.trace("After compaction, scan time for {} rows {} ms", NUM_ROWS,
    +            TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        if (count != NUM_ROWS) {
    +          throw new IllegalStateException(String.format("After compaction, number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +        }
    +
    +      } catch (TableNotFoundException ex) {
    +        throw new IllegalStateException("test failed, table " + tableName + " does not exist", ex);
    +      } catch (AccumuloSecurityException ex) {
    +        throw new IllegalStateException("test failed, could not add iterator due to security exception", ex);
    +      } catch (AccumuloException ex) {
    +        // test cancels compaction on complete, so ignore it as an exception.
    +        if (!ex.getMessage().contains("Compaction canceled")) {
    --- End diff --
    
    This stack element is thrown: 
    
    Caused by: org.apache.accumulo.core.client.AccumuloException: Compaction canceled.  
    
    Did not want to swallow all AccumuloExceptions - just the Compaction cancel. (Not sure why this is an exception when requested, but it probably is a rare event and does provide history in the log.)


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] accumulo pull request #204: ACCUMULO-4574 Modified TableOperations online to...

Posted by joshelser <gi...@git.apache.org>.
Github user joshelser commented on a diff in the pull request:

    https://github.com/apache/accumulo/pull/204#discussion_r97357463
  
    --- Diff: test/src/test/java/org/apache/accumulo/test/functional/TableChangeStateIT.java ---
    @@ -0,0 +1,549 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one or more
    + * contributor license agreements.  See the NOTICE file distributed with
    + * this work for additional information regarding copyright ownership.
    + * The ASF licenses this file to You under the Apache License, Version 2.0
    + * (the "License"); you may not use this file except in compliance with
    + * the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.accumulo.test.functional;
    +
    +import org.apache.accumulo.core.Constants;
    +import org.apache.accumulo.core.client.AccumuloException;
    +import org.apache.accumulo.core.client.AccumuloSecurityException;
    +import org.apache.accumulo.core.client.BatchWriter;
    +import org.apache.accumulo.core.client.BatchWriterConfig;
    +import org.apache.accumulo.core.client.Connector;
    +import org.apache.accumulo.core.client.IteratorSetting;
    +import org.apache.accumulo.core.client.Scanner;
    +import org.apache.accumulo.core.client.TableExistsException;
    +import org.apache.accumulo.core.client.TableNotFoundException;
    +import org.apache.accumulo.core.client.impl.Tables;
    +import org.apache.accumulo.core.data.Key;
    +import org.apache.accumulo.core.data.Mutation;
    +import org.apache.accumulo.core.data.Value;
    +import org.apache.accumulo.core.master.state.tables.TableState;
    +import org.apache.accumulo.core.security.Authorizations;
    +import org.apache.accumulo.core.zookeeper.ZooUtil;
    +import org.apache.accumulo.fate.zookeeper.ZooCache;
    +import org.apache.accumulo.fate.zookeeper.ZooReader;
    +import org.apache.hadoop.io.Text;
    +import org.junit.Test;
    +import org.slf4j.Logger;
    +import org.slf4j.LoggerFactory;
    +
    +import java.util.ArrayList;
    +import java.util.Collections;
    +import java.util.List;
    +import java.util.Map;
    +import java.util.concurrent.TimeUnit;
    +import java.util.concurrent.atomic.AtomicBoolean;
    +
    +import static java.nio.charset.StandardCharsets.UTF_8;
    +import static org.junit.Assert.assertEquals;
    +import static org.junit.Assert.assertFalse;
    +import static org.junit.Assert.assertTrue;
    +
    +/**
    + * ACCUMULO-4574. Test to verify that changing table state to online / offline {@link org.apache.accumulo.core.client.admin.TableOperations#online(String)} when
    + * the table is already in that state returns without blocking.
    + */
    +public class TableChangeStateIT extends ConfigurableMacIT {
    +
    +  private static final Logger log = LoggerFactory.getLogger(TableChangeStateIT.class);
    +
    +  private static final int NUM_ROWS = 1000;
    +
    +  @Override
    +  protected int defaultTimeoutSeconds() {
    +    return 4 * 60;
    +  }
    +
    +  /**
    +   * Validate that {@code TableOperations} online operation does not block when table is already online and fate transaction lock is held by other operations.
    +   * The test creates, populates a table and then runs a compaction with a slow iterator so that operation takes long enough to simulate the condition. After
    +   * the online operation while compaction is running completes, the test is complete and the compaction is canceled so that other tests can run.
    +   *
    +   * @throws Exception
    +   *           any exception is a test failure.
    +   */
    +  @Test
    +  public void changeTableStateTest() throws Exception {
    +
    +    Connector connector = getConnector();
    +    String tableName = getUniqueNames(1)[0];
    +
    +    createData(tableName);
    +
    +    assertEquals("verify table online after created", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status = onLine(tableName);
    +
    +    log.trace("Online 1 in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    assertFalse("verify no exception thrown changing state", status.hasError());
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // verify that offline then online functions as expected.
    +
    +    connector.tableOperations().offline(tableName, true);
    +    assertEquals("verify table is offline", TableState.OFFLINE, getTableState(connector, tableName));
    +
    +    OnlineStatus status2 = onLine(tableName);
    +
    +    log.trace("Online 2 in {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status2.hasError()) {
    +      log.debug("Online failed with exception", status2.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status2.hasError());
    +    assertEquals("verify table is back online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    // launch a full table compaction with the slow iterator to ensure table lock is acquired and held by the compaction
    +    Thread compactThread = new Thread(new SlowCompactionRunner(tableName));
    +    compactThread.start();
    +
    +    assertTrue("verify that compaction running and fate transaction exists", blockUntilCompactionRunning());
    +
    +    // try to set online while fate transaction is in progress - before ACCUMULO-4574 this would block
    +    OnlineStatus status3 = onLine(tableName);
    +    log.trace("Online while compacting in {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    if (status3.hasError()) {
    +      log.debug("Online failed with exception", status3.getException());
    +    }
    +
    +    assertFalse("verify no exception thrown changing state", status3.hasError());
    +    assertTrue("online should take less time than expected compaction time", status3.runningTime() < TimeUnit.NANOSECONDS.convert(2, TimeUnit.SECONDS));
    +    assertEquals("verify table is still online", TableState.ONLINE, getTableState(connector, tableName));
    +
    +    assertTrue("verify compaction still running and fate transaction still exists", blockUntilCompactionRunning());
    +
    +    // test complete, cancel compaction and move on.
    +    connector.tableOperations().cancelCompaction(tableName);
    +
    +    log.debug("Success: Timing results for online commands.");
    +    log.debug("Time for unblocked online {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for online when offline {} ms", TimeUnit.MILLISECONDS.convert(status2.runningTime(), TimeUnit.NANOSECONDS));
    +    log.debug("Time for blocked online {} ms", TimeUnit.MILLISECONDS.convert(status3.runningTime(), TimeUnit.NANOSECONDS));
    +
    +    compactThread.join();
    +
    +  }
    +
    +  /**
    +   * Blocks current thread until compaction is running.
    +   *
    +   * @return true if compaction and associate fate found.
    +   */
    +  private boolean blockUntilCompactionRunning() {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      int runningCompactions = 0;
    +
    +      List<String> tservers = connector.instanceOperations().getTabletServers();
    +
    +      /*
    +       * wait for compaction to start - The compaction will acquire a fate transaction lock that used to block a subsequent online command while the fate
    +       * transaction lock was held.
    +       */
    +      while (runningCompactions == 0) {
    +
    +        try {
    +
    +          for (String tserver : tservers) {
    +            runningCompactions += connector.instanceOperations().getActiveCompactions(tserver).size();
    +            log.trace("tserver {}, running compactions {}", tservers, runningCompactions);
    +          }
    +
    +        } catch (AccumuloSecurityException | AccumuloException ex) {
    +          throw new IllegalStateException("failed to get active compactions, test fails.", ex);
    +        }
    +
    +        try {
    +          Thread.sleep(250);
    +        } catch (InterruptedException ex) {
    +          // reassert interrupt
    +          Thread.currentThread().interrupt();
    +        }
    +      }
    +
    +      // Validate that there is a compaction fate transaction - otherwise test is invalid.
    +      return findFate();
    +
    +    } catch (AccumuloSecurityException | AccumuloException ex) {
    +      throw new IllegalStateException("test failed waiting for compaction to start", ex);
    +    }
    +  }
    +
    +  /**
    +   * Checks fates in zookeeper looking for transaction associated with a compaction as a double check that the test will be valid because the running compaction
    +   * does have a fate transaction lock.
    +   *
    +   * @return true if corresponding fate transaction found, false otherwise
    +   * @throws AccumuloException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   * @throws AccumuloSecurityException
    +   *           from {@link ConfigurableMacIT#getConnector}
    +   */
    +  private boolean findFate() throws AccumuloException, AccumuloSecurityException {
    +
    +    Connector connector = getConnector();
    +
    +    String zPath = ZooUtil.getRoot(connector.getInstance()) + Constants.ZFATE;
    +    ZooReader zooReader = new ZooReader(connector.getInstance().getZooKeepers(), connector.getInstance().getZooKeepersSessionTimeOut());
    +    ZooCache zooCache = new ZooCache(zooReader, null);
    +
    +    List<String> lockedIds = zooCache.getChildren(zPath);
    +
    +    for (String fateId : lockedIds) {
    +
    +      List<String> lockNodes = zooCache.getChildren(zPath + "/" + fateId);
    +      lockNodes = new ArrayList<>(lockNodes);
    +      Collections.sort(lockNodes);
    +
    +      for (String node : lockNodes) {
    +
    +        byte[] data = zooCache.get(zPath + "/" + fateId + "/" + node);
    +        String lda[] = new String(data, UTF_8).split(":");
    +
    +        for (String fateString : lda) {
    +
    +          log.trace("Lock: {}:{}: {}", fateId, node, lda);
    +
    +          if (node.contains("prop_debug") && fateString.contains("CompactRange")) {
    +            // found fate associated with table compaction.
    +            log.trace("FOUND - Lock: {}:{}: {}", fateId, node, lda);
    +            return Boolean.TRUE;
    +          }
    +        }
    +      }
    +    }
    +
    +    // did not find appropriate fate transaction for compaction.
    +    return Boolean.FALSE;
    +  }
    +
    +  /**
    +   * Returns the current table state (ONLINE, OFFLINE,...) of named table.
    +   *
    +   * @param connector
    +   *          connector to Accumulo instance
    +   * @param tableName
    +   *          the table name
    +   * @return the current table state
    +   * @throws TableNotFoundException
    +   *           if table does not exist
    +   */
    +  private TableState getTableState(Connector connector, String tableName) throws TableNotFoundException {
    +
    +    String tableId = Tables.getTableId(connector.getInstance(), tableName);
    +
    +    TableState tstate = Tables.getTableState(connector.getInstance(), tableId);
    +
    +    log.trace("tableName: '{}': tableId {}, current state: {}", tableName, tableId, tstate);
    +
    +    return tstate;
    +  }
    +
    +  /**
    +   * Executes TableOperations online command in a separate thread and blocks the current thread until command is complete. The status and time to complete is
    +   * returned in {@code OnlineStatus}
    +   *
    +   * @param tableName
    +   *          the table to set online.
    +   * @return status that can be used to determine if running and timing information when complete.
    +   */
    +
    +  private OnlineStatus onLine(String tableName) {
    +
    +    OnLineThread onLineCmd = new OnLineThread(tableName);
    +    Thread onLineThread = new Thread(onLineCmd);
    +    onLineThread.start();
    +
    +    int count = 0;
    +
    +    while (onLineCmd.getStatus().isRunning()) {
    +
    +      if ((count++ % 10) == 0) {
    +        log.trace("online operation blocked, waiting for it to complete.");
    +      }
    +
    +      try {
    +        Thread.sleep(1000);
    +      } catch (InterruptedException ex) {
    +        // reassert interrupt
    +        Thread.currentThread().interrupt();
    +      }
    +    }
    +
    +    return onLineCmd.getStatus();
    +  }
    +
    +  /**
    +   * Create the provided table and populate with some data using a batch writer. The table is scanned to ensure it was populated as expected.
    +   *
    +   * @param tableName
    +   *          the name of the table
    +   */
    +  private void createData(final String tableName) {
    +
    +    try {
    +
    +      Connector connector = getConnector();
    +
    +      // create table.
    +      connector.tableOperations().create(tableName);
    +      BatchWriter bw = connector.createBatchWriter(tableName, new BatchWriterConfig());
    +
    +      // populate
    +      for (int i = 0; i < NUM_ROWS; i++) {
    +        Mutation m = new Mutation(new Text(String.format("%05d", i)));
    +        m.put(new Text("col" + Integer.toString((i % 3) + 1)), new Text("qual"), new Value("junk".getBytes(UTF_8)));
    +        bw.addMutation(m);
    +      }
    +      bw.close();
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +      int count = 0;
    +      for (Map.Entry<Key,Value> elt : scanner) {
    +        String expected = String.format("%05d", count);
    +        assert (elt.getKey().getRow().toString().equals(expected));
    +        count++;
    +      }
    +
    +      log.trace("Scan time for {} rows {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +      scanner.close();
    +
    +      if (count != NUM_ROWS) {
    +        throw new IllegalStateException(String.format("Number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +      }
    +    } catch (AccumuloException | AccumuloSecurityException | TableNotFoundException | TableExistsException ex) {
    +      throw new IllegalStateException("Create data failed with exception", ex);
    +    }
    +  }
    +
    +  /**
    +   * Provides timing information and online command status results for online command executed asynchronously. The
    +   */
    +  private static class OnlineStatus {
    +
    +    private long started = 0L;
    +    private long completed = 0L;
    +
    +    // set to true - even before running so that we can block on isRunning before thread may have started.
    +    private final AtomicBoolean isRunning = new AtomicBoolean(Boolean.TRUE);
    +    private Boolean hasError = Boolean.FALSE;
    +    private Exception exception = null;
    +
    +    /**
    +     * Returns true until operation has completed.
    +     *
    +     * @return false when operation has been set complete.
    +     */
    +    Boolean isRunning() {
    +      return isRunning.get();
    +    }
    +
    +    /**
    +     * start timing operation.
    +     */
    +    void setRunning() {
    +      isRunning.set(Boolean.TRUE);
    +      started = System.nanoTime();
    +    }
    +
    +    /**
    +     * stop timing and set completion flag.
    +     */
    +    void setComplete() {
    +      completed = System.nanoTime();
    +      isRunning.set(Boolean.FALSE);
    +    }
    +
    +    /**
    +     * Returns true if setError was called to complete operation.
    +     *
    +     * @return true if an error was set, false otherwise.
    +     */
    +    boolean hasError() {
    +      return hasError;
    +    }
    +
    +    /**
    +     * Returns exception if set by {@code setError}.
    +     *
    +     * @return the exception set with {@code setError} or null if not set.
    +     */
    +    public Exception getException() {
    +      return exception;
    +    }
    +
    +    /**
    +     * Marks the operation as complete, stops timing the operation, with error status and exception that caused the failure.
    +     *
    +     * @param ex
    +     *          the exception that caused failure.
    +     */
    +    public void setError(Exception ex) {
    +      hasError = Boolean.TRUE;
    +      exception = ex;
    +      setComplete();
    +    }
    +
    +    /**
    +     * @return running time in nanoseconds.
    +     */
    +    long runningTime() {
    +      return completed - started;
    +    }
    +  }
    +
    +  /**
    +   * Run online operation in a separate thread. The operation status can be tracked with {@link OnlineStatus} returned by {@link OnLineThread#getStatus}. The
    +   * operation timing is started when run is called. If an exception occurs, it is available in the status.
    +   */
    +  private class OnLineThread implements Runnable {
    +
    +    final String tableName;
    +    final OnlineStatus status;
    +
    +    /**
    +     * Create an instance of this class to set the provided table online.
    +     *
    +     * @param tableName
    +     *          The table name that will be set online.
    +     */
    +    OnLineThread(final String tableName) {
    +      this.tableName = tableName;
    +      this.status = new OnlineStatus();
    +    }
    +
    +    /**
    +     * Update status for timing and execute the online operation.
    +     */
    +    @Override
    +    public void run() {
    +
    +      status.setRunning();
    +
    +      try {
    +
    +        log.trace("Setting {} online", tableName);
    +
    +        getConnector().tableOperations().online(tableName, true);
    +
    +        // stop timing
    +        status.setComplete();
    +        log.trace("Online completed in {} ms", TimeUnit.MILLISECONDS.convert(status.runningTime(), TimeUnit.NANOSECONDS));
    +
    +      } catch (Exception ex) {
    +        // set error in status with this exception.
    +        status.setError(ex);
    +      }
    +
    +    }
    +
    +    /**
    +     * Provide OnlineStatus of this operation to determine when complete and timing information
    +     *
    +     * @return OnlineStatus to determine when complete and timing information
    +     */
    +    public OnlineStatus getStatus() {
    +      return status;
    +    }
    +
    +  }
    +
    +  /**
    +   * Instance to create / run a compaction using a slow iterator.
    +   */
    +  private class SlowCompactionRunner implements Runnable {
    +
    +    private final String tableName;
    +
    +    /**
    +     * Create an instance of this class.
    +     *
    +     * @param tableName
    +     *          the name of the table that will be compacted with the slow iterator.
    +     */
    +    SlowCompactionRunner(final String tableName) {
    +      this.tableName = tableName;
    +    }
    +
    +    @Override
    +    public void run() {
    +
    +      long startTimestamp = System.nanoTime();
    +
    +      IteratorSetting slow = new IteratorSetting(30, "slow", SlowIterator.class);
    +      SlowIterator.setSleepTime(slow, 100);
    +
    +      List<IteratorSetting> compactIterators = new ArrayList<>();
    +      compactIterators.add(slow);
    +
    +      log.trace("Slow iterator {}", slow.toString());
    +
    +      try {
    +
    +        Connector connector = getConnector();
    +
    +        log.trace("Start compaction");
    +
    +        connector.tableOperations().compact(tableName, new Text("0"), new Text("z"), compactIterators, true, true);
    +
    +        log.trace("Compaction wait is complete");
    +
    +        log.trace("Slow compaction of {} rows took {} ms", NUM_ROWS, TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        // validate that number of rows matches expected.
    +
    +        startTimestamp = System.nanoTime();
    +
    +        // validate expected data created and exists in table.
    +
    +        Scanner scanner = connector.createScanner(tableName, Authorizations.EMPTY);
    +
    +        int count = 0;
    +        for (Map.Entry<Key,Value> elt : scanner) {
    +          String expected = String.format("%05d", count);
    +          assert (elt.getKey().getRow().toString().equals(expected));
    +          count++;
    +        }
    +
    +        log.trace("After compaction, scan time for {} rows {} ms", NUM_ROWS,
    +            TimeUnit.MILLISECONDS.convert((System.nanoTime() - startTimestamp), TimeUnit.NANOSECONDS));
    +
    +        if (count != NUM_ROWS) {
    +          throw new IllegalStateException(String.format("After compaction, number of rows %1$d does not match expected %2$d", count, NUM_ROWS));
    +        }
    +
    +      } catch (TableNotFoundException ex) {
    +        throw new IllegalStateException("test failed, table " + tableName + " does not exist", ex);
    +      } catch (AccumuloSecurityException ex) {
    +        throw new IllegalStateException("test failed, could not add iterator due to security exception", ex);
    +      } catch (AccumuloException ex) {
    +        // test cancels compaction on complete, so ignore it as an exception.
    +        if (!ex.getMessage().contains("Compaction canceled")) {
    --- End diff --
    
    Do we not get a named exception when the compaction is cancelled?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---