17
17
*/
18
18
package org .apache .hadoop .hbase .io .hfile ;
19
19
20
+ import com .google .errorprone .annotations .RestrictedApi ;
20
21
import java .util .Map ;
21
22
import java .util .concurrent .ConcurrentSkipListMap ;
22
23
import java .util .concurrent .Future ;
23
24
import java .util .concurrent .RejectedExecutionException ;
24
25
import java .util .concurrent .ScheduledExecutorService ;
26
+ import java .util .concurrent .ScheduledFuture ;
25
27
import java .util .concurrent .ScheduledThreadPoolExecutor ;
26
28
import java .util .concurrent .ThreadFactory ;
27
29
import java .util .concurrent .ThreadLocalRandom ;
28
30
import java .util .concurrent .TimeUnit ;
31
+ import java .util .concurrent .atomic .AtomicBoolean ;
29
32
import java .util .regex .Pattern ;
30
33
import org .apache .hadoop .conf .Configuration ;
31
34
import org .apache .hadoop .fs .Path ;
41
44
public final class PrefetchExecutor {
42
45
43
46
private static final Logger LOG = LoggerFactory .getLogger (PrefetchExecutor .class );
47
+ /** Wait time in miliseconds before executing prefetch */
48
+ public static final String PREFETCH_DELAY = "hbase.hfile.prefetch.delay" ;
49
+ public static final String PREFETCH_DELAY_VARIATION = "hbase.hfile.prefetch.delay.variation" ;
50
+ public static final float PREFETCH_DELAY_VARIATION_DEFAULT_VALUE = 0.2f ;
44
51
45
52
/** Futures for tracking block prefetch activity */
46
53
private static final Map <Path , Future <?>> prefetchFutures = new ConcurrentSkipListMap <>();
54
+ /** Runnables for resetting the prefetch activity */
55
+ private static final Map <Path , Runnable > prefetchRunnable = new ConcurrentSkipListMap <>();
47
56
/** Executor pool shared among all HFiles for block prefetch */
48
57
private static final ScheduledExecutorService prefetchExecutorPool ;
49
58
/** Delay before beginning prefetch */
50
- private static final int prefetchDelayMillis ;
59
+ private static int prefetchDelayMillis ;
51
60
/** Variation in prefetch delay times, to mitigate stampedes */
52
- private static final float prefetchDelayVariation ;
61
+ private static float prefetchDelayVariation ;
53
62
static {
54
63
// Consider doing this on demand with a configuration passed in rather
55
64
// than in a static initializer.
56
65
Configuration conf = HBaseConfiguration .create ();
57
66
// 1s here for tests, consider 30s in hbase-default.xml
58
67
// Set to 0 for no delay
59
- prefetchDelayMillis = conf .getInt ("hbase.hfile.prefetch.delay" , 1000 );
60
- prefetchDelayVariation = conf .getFloat ("hbase.hfile.prefetch.delay.variation" , 0.2f );
68
+ prefetchDelayMillis = conf .getInt (PREFETCH_DELAY , 1000 );
69
+ prefetchDelayVariation =
70
+ conf .getFloat (PREFETCH_DELAY_VARIATION , PREFETCH_DELAY_VARIATION_DEFAULT_VALUE );
61
71
int prefetchThreads = conf .getInt ("hbase.hfile.thread.prefetch" , 4 );
62
72
prefetchExecutorPool = new ScheduledThreadPoolExecutor (prefetchThreads , new ThreadFactory () {
63
73
@ Override
@@ -95,15 +105,18 @@ public static void request(Path path, Runnable runnable) {
95
105
final Future <?> future =
96
106
prefetchExecutorPool .schedule (tracedRunnable , delay , TimeUnit .MILLISECONDS );
97
107
prefetchFutures .put (path , future );
108
+ prefetchRunnable .put (path , runnable );
98
109
} catch (RejectedExecutionException e ) {
99
110
prefetchFutures .remove (path );
111
+ prefetchRunnable .remove (path );
100
112
LOG .warn ("Prefetch request rejected for {}" , path );
101
113
}
102
114
}
103
115
}
104
116
105
117
public static void complete (Path path ) {
106
118
prefetchFutures .remove (path );
119
+ prefetchRunnable .remove (path );
107
120
if (LOG .isDebugEnabled ()) {
108
121
LOG .debug ("Prefetch completed for {}" , path .getName ());
109
122
}
@@ -115,23 +128,85 @@ public static void cancel(Path path) {
115
128
// ok to race with other cancellation attempts
116
129
future .cancel (true );
117
130
prefetchFutures .remove (path );
131
+ prefetchRunnable .remove (path );
118
132
LOG .debug ("Prefetch cancelled for {}" , path );
119
133
}
120
134
}
121
135
122
- public static boolean isCompleted (Path path ) {
136
+ public static void interrupt (Path path ) {
123
137
Future <?> future = prefetchFutures .get (path );
124
138
if (future != null ) {
125
- return future .isDone ();
139
+ prefetchFutures .remove (path );
140
+ // ok to race with other cancellation attempts
141
+ future .cancel (true );
142
+ LOG .debug ("Prefetch cancelled for {}" , path );
126
143
}
127
- return true ;
128
144
}
129
145
130
146
private PrefetchExecutor () {
131
147
}
132
148
149
+ public static boolean isCompleted (Path path ) {
150
+ Future <?> future = prefetchFutures .get (path );
151
+ if (future != null ) {
152
+ return future .isDone ();
153
+ }
154
+ return true ;
155
+ }
156
+
133
157
/* Visible for testing only */
158
+ @ RestrictedApi (explanation = "Should only be called in tests" , link = "" ,
159
+ allowedOnPath = ".*/src/test/.*" )
134
160
static ScheduledExecutorService getExecutorPool () {
135
161
return prefetchExecutorPool ;
136
162
}
163
+
164
+ @ RestrictedApi (explanation = "Should only be called in tests" , link = "" ,
165
+ allowedOnPath = ".*/src/test/.*" )
166
+ static Map <Path , Future <?>> getPrefetchFutures () {
167
+ return prefetchFutures ;
168
+ }
169
+
170
+ @ RestrictedApi (explanation = "Should only be called in tests" , link = "" ,
171
+ allowedOnPath = ".*/src/test/.*" )
172
+ static Map <Path , Runnable > getPrefetchRunnable () {
173
+ return prefetchRunnable ;
174
+ }
175
+
176
+ static boolean isPrefetchStarted () {
177
+ AtomicBoolean prefetchStarted = new AtomicBoolean (false );
178
+ for (Map .Entry <Path , Future <?>> entry : prefetchFutures .entrySet ()) {
179
+ Path k = entry .getKey ();
180
+ Future <?> v = entry .getValue ();
181
+ ScheduledFuture sf = (ScheduledFuture ) prefetchFutures .get (k );
182
+ long waitTime = sf .getDelay (TimeUnit .MILLISECONDS );
183
+ if (waitTime < 0 ) {
184
+ // At this point prefetch is started
185
+ prefetchStarted .set (true );
186
+ break ;
187
+ }
188
+ }
189
+ return prefetchStarted .get ();
190
+ }
191
+
192
+ public static int getPrefetchDelay () {
193
+ return prefetchDelayMillis ;
194
+ }
195
+
196
+ public static void loadConfiguration (Configuration conf ) {
197
+ prefetchDelayMillis = conf .getInt (PREFETCH_DELAY , 1000 );
198
+ prefetchDelayVariation =
199
+ conf .getFloat (PREFETCH_DELAY_VARIATION , PREFETCH_DELAY_VARIATION_DEFAULT_VALUE );
200
+ prefetchFutures .forEach ((k , v ) -> {
201
+ ScheduledFuture sf = (ScheduledFuture ) prefetchFutures .get (k );
202
+ if (!(sf .getDelay (TimeUnit .MILLISECONDS ) > 0 )) {
203
+ // the thread is still pending delay expiration and has not started to run yet, so can be
204
+ // re-scheduled at no cost.
205
+ interrupt (k );
206
+ request (k , prefetchRunnable .get (k ));
207
+ }
208
+ LOG .debug ("Reset called on Prefetch of file {} with delay {}, delay variation {}" , k ,
209
+ prefetchDelayMillis , prefetchDelayVariation );
210
+ });
211
+ }
137
212
}
0 commit comments