001/** 002 * Licensed to the Apache Software Foundation (ASF) under one 003 * or more contributor license agreements. See the NOTICE file 004 * distributed with this work for additional information 005 * regarding copyright ownership. The ASF licenses this file 006 * to you under the Apache License, Version 2.0 (the 007 * "License"); you may not use this file except in compliance 008 * with the License. You may obtain a copy of the License at 009 * 010 * http://www.apache.org/licenses/LICENSE-2.0 011 * 012 * Unless required by applicable law or agreed to in writing, software 013 * distributed under the License is distributed on an "AS IS" BASIS, 014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 015 * See the License for the specific language governing permissions and 016 * limitations under the License. 017 */ 018package org.apache.hadoop.hbase.client; 019 020import java.io.Closeable; 021import java.util.Collections; 022import java.util.List; 023import java.util.concurrent.CompletableFuture; 024import java.util.concurrent.TimeUnit; 025import org.apache.hadoop.conf.Configuration; 026import org.apache.hadoop.hbase.TableName; 027import org.apache.yetus.audience.InterfaceAudience; 028 029import org.apache.hbase.thirdparty.com.google.common.collect.Iterables; 030 031/** 032 * Used to communicate with a single HBase table in batches. Obtain an instance from a 033 * {@link AsyncConnection} and call {@link #close()} afterwards. 034 * <p> 035 * The implementation is required to be thread safe. 036 */ 037@InterfaceAudience.Public 038public interface AsyncBufferedMutator extends Closeable { 039 040 /** 041 * Gets the fully qualified table name instance of the table that this 042 * {@code AsyncBufferedMutator} writes to. 043 */ 044 TableName getName(); 045 046 /** 047 * Returns the {@link org.apache.hadoop.conf.Configuration} object used by this instance. 048 * <p> 049 * The reference returned is not a copy, so any change made to it will affect this instance. 050 */ 051 Configuration getConfiguration(); 052 053 /** 054 * Sends a {@link Mutation} to the table. The mutations will be buffered and sent over the wire as 055 * part of a batch. Currently only supports {@link Put} and {@link Delete} mutations. 056 * @param mutation The data to send. 057 */ 058 default CompletableFuture<Void> mutate(Mutation mutation) { 059 return Iterables.getOnlyElement(mutate(Collections.singletonList(mutation))); 060 } 061 062 /** 063 * Send some {@link Mutation}s to the table. The mutations will be buffered and sent over the wire 064 * as part of a batch. There is no guarantee of sending entire content of {@code mutations} in a 065 * single batch, the implementations are free to break it up according to the write buffer 066 * capacity. 067 * @param mutations The data to send. 068 */ 069 List<CompletableFuture<Void>> mutate(List<? extends Mutation> mutations); 070 071 /** 072 * Executes all the buffered, asynchronous operations. 073 */ 074 void flush(); 075 076 /** 077 * Performs a {@link #flush()} and releases any resources held. 078 */ 079 @Override 080 void close(); 081 082 /** 083 * Returns the maximum size in bytes of the write buffer. 084 * <p> 085 * The default value comes from the configuration parameter {@code hbase.client.write.buffer}. 086 * @return The size of the write buffer in bytes. 087 */ 088 long getWriteBufferSize(); 089 090 /** 091 * Returns the periodical flush interval, 0 means disabled. 092 */ 093 default long getPeriodicalFlushTimeout(TimeUnit unit) { 094 throw new UnsupportedOperationException("Not implemented"); 095 } 096}