From cf58fb08aa67559671ae661e16a81151ca978c87 Mon Sep 17 00:00:00 2001 From: =?UTF-8?q?D=C3=A1vid=20Karnok?= Date: Sat, 11 Feb 2017 15:02:41 +0100 Subject: [PATCH] 1.x: add Single.merge(Obs), Obs.flatMapSingle & flatMapCompletable --- src/main/java/rx/Observable.java | 136 +++ src/main/java/rx/Single.java | 91 ++ .../OnSubscribeFlatMapCompletable.java | 214 +++++ .../operators/OnSubscribeFlatMapSingle.java | 333 ++++++++ src/test/java/rx/CompletableTest.java | 4 +- .../OnSubscribeFlatMapCompletableTest.java | 557 +++++++++++++ .../OnSubscribeFlatMapSingleTest.java | 782 ++++++++++++++++++ 7 files changed, 2115 insertions(+), 2 deletions(-) create mode 100644 src/main/java/rx/internal/operators/OnSubscribeFlatMapCompletable.java create mode 100644 src/main/java/rx/internal/operators/OnSubscribeFlatMapSingle.java create mode 100644 src/test/java/rx/internal/operators/OnSubscribeFlatMapCompletableTest.java create mode 100644 src/test/java/rx/internal/operators/OnSubscribeFlatMapSingleTest.java diff --git a/src/main/java/rx/Observable.java b/src/main/java/rx/Observable.java index bdc59e33ce..5978412fc7 100644 --- a/src/main/java/rx/Observable.java +++ b/src/main/java/rx/Observable.java @@ -6973,6 +6973,74 @@ public final Observable flatMap(final Func1(collectionSelector, resultSelector)), maxConcurrent); } + /** + * Maps all upstream values to Completables and runs them together until the upstream + * and all inner Completables complete normally. + *
+ *
Backpressure:
+ *
The operator consumes items from upstream in an unbounded manner and ignores downstream backpressure + * as it doesn't emit items but only terminal event.
+ *
Scheduler:
+ *
{@code flatMapCompletable} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param mapper the function that receives an upstream value and turns it into a Completable + * to be merged. + * @return the new Observable instance + * @see #flatMapCompletable(Func1, boolean, int) + * @since 1.2.7 - experimental + */ + @Experimental + public final Observable flatMapCompletable(Func1 mapper) { + return flatMapCompletable(mapper, false, Integer.MAX_VALUE); + } + + /** + * Maps all upstream values to Completables and runs them together, optionally delaying any errors, until the upstream + * and all inner Completables terminate. + *
+ *
Backpressure:
+ *
The operator consumes items from upstream in an unbounded manner and ignores downstream backpressure + * as it doesn't emit items but only terminal event.
+ *
Scheduler:
+ *
{@code flatMapCompletable} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param mapper the function that receives an upstream value and turns it into a Completable + * to be merged. + * @param delayErrors if true, errors from the upstream and from the inner Completables get delayed till + * the all of them terminate. + * @return the new Observable instance + * @since 1.2.7 - experimental + * @see #flatMapCompletable(Func1, boolean, int) + */ + @Experimental + public final Observable flatMapCompletable(Func1 mapper, boolean delayErrors) { + return flatMapCompletable(mapper, delayErrors, Integer.MAX_VALUE); + } + + /** + * Maps upstream values to Completables and runs up to the given number of them together at a time, + * optionally delaying any errors, until the upstream and all inner Completables terminate. + *
+ *
Backpressure:
+ *
The operator consumes at most maxConcurrent items from upstream and one-by-one after as the inner + * Completables terminate. The operator ignores downstream backpressure as it doesn't emit items but + * only the terminal event.
+ *
Scheduler:
+ *
{@code flatMapCompletable} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param mapper the function that receives an upstream value and turns it into a Completable + * to be merged. + * @param delayErrors if true, errors from the upstream and from the inner Completables get delayed till + * the all of them terminate. + * @param maxConcurrency the maximum number of inner Completables to run at a time + * @return the new Observable instance + * @since 1.2.7 - experimental + */ + @Experimental + public final Observable flatMapCompletable(Func1 mapper, boolean delayErrors, int maxConcurrency) { + return unsafeCreate(new OnSubscribeFlatMapCompletable(this, mapper, delayErrors, maxConcurrency)); + } + /** * Returns an Observable that merges each item emitted by the source Observable with the values in an * Iterable corresponding to that item that is generated by a selector. @@ -7106,6 +7174,74 @@ public final Observable flatMapIterable(Func1)flatMap(OperatorMapPair.convertSelector(collectionSelector), resultSelector, maxConcurrent); } + /** + * Maps all upstream values to Singles and runs them together until the upstream + * and all inner Singles complete normally. + *
+ *
Backpressure:
+ *
The operator consumes items from upstream in an unbounded manner and honors downstream backpressure.
+ *
Scheduler:
+ *
{@code flatMapSingle} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param the value type of the inner Singles and the resulting Observable + * @param mapper the function that receives an upstream value and turns it into a Single + * to be merged. + * @return the new Observable instance + * @see #flatMapSingle(Func1, boolean, int) + * @since 1.2.7 - experimental + */ + @Experimental + public final Observable flatMapSingle(Func1> mapper) { + return flatMapSingle(mapper, false, Integer.MAX_VALUE); + } + + /** + * Maps all upstream values to Singles and runs them together, optionally delaying any errors, until the upstream + * and all inner Singles terminate. + *
+ *
Backpressure:
+ *
The operator consumes items from upstream in an unbounded manner and honors downstream backpressure.
+ *
Scheduler:
+ *
{@code flatMapSingle} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param the value type of the inner Singles and the resulting Observable + * @param mapper the function that receives an upstream value and turns it into a Single + * to be merged. + * @param delayErrors if true, errors from the upstream and from the inner Singles get delayed till + * the all of them terminate. + * @return the new Observable instance + * @since 1.2.7 - experimental + * @see #flatMapSingle(Func1, boolean, int) + */ + @Experimental + public final Observable flatMapSingle(Func1> mapper, boolean delayErrors) { + return flatMapSingle(mapper, delayErrors, Integer.MAX_VALUE); + } + + /** + * Maps upstream values to Singles and runs up to the given number of them together at a time, + * optionally delaying any errors, until the upstream and all inner Singles terminate. + *
+ *
Backpressure:
+ *
The operator consumes at most maxConcurrent items from upstream and one-by-one after as the inner + * Singles terminate. The operator honors downstream backpressure.
+ *
Scheduler:
+ *
{@code flatMapSingle} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param the value type of the inner Singles and the resulting Observable + * @param mapper the function that receives an upstream value and turns it into a Single + * to be merged. + * @param delayErrors if true, errors from the upstream and from the inner Singles get delayed till + * the all of them terminate. + * @param maxConcurrency the maximum number of inner Singles to run at a time + * @return the new Observable instance + * @since 1.2.7 - experimental + */ + @Experimental + public final Observable flatMapSingle(Func1> mapper, boolean delayErrors, int maxConcurrency) { + return unsafeCreate(new OnSubscribeFlatMapSingle(this, mapper, delayErrors, maxConcurrency)); + } + /** * Subscribes to the {@link Observable} and receives notifications for each element. *

diff --git a/src/main/java/rx/Single.java b/src/main/java/rx/Single.java index 709b709f41..a1614137f6 100644 --- a/src/main/java/rx/Single.java +++ b/src/main/java/rx/Single.java @@ -930,6 +930,97 @@ public static Observable merge(Single t1, Single + *

Backpressure:
+ *
The operator consumes items from the Observable in an unbounded manner and honors downstream backpressure.
+ *
Scheduler:
+ *
{@code merge} does not operate by default on a particular {@link Scheduler}.
+ * + * @param the value type of the inner Singles and the resulting Observable + * @param sources the Observable that emits Singles to be merged + * @return the new Observable instance + * @see #merge(Observable, int) + * @see #mergeDelayError(Observable) + * @see #mergeDelayError(Observable, int) + * @since 1.2.7 - experimental + */ + @Experimental + public static Observable merge(Observable> sources) { + return merge(sources, Integer.MAX_VALUE); + } + + /** + * Merges the Singles emitted by the Observable and runs up to the given number of them together at a time, + * until the Observable and all inner Singles terminate. + *
+ *
Backpressure:
+ *
The operator consumes at most maxConcurrent items from the Observable and one-by-one after as the inner + * Singles terminate. The operator ignores downstream backpressure as it doesn't emit items but + * only the terminal event.
+ *
Scheduler:
+ *
{@code flatMapSingle} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param the value type of the inner Singles and the resulting Observable + * @param sources the Observable that emits Singles to be merged + * @param maxConcurrency the maximum number of inner Singles to run at a time + * @return the new Observable instance + * @since 1.2.7 - experimental + */ + @SuppressWarnings({ "unchecked", "rawtypes" }) + @Experimental + public static Observable merge(Observable> sources, int maxConcurrency) { + return sources.flatMapSingle((Func1)UtilityFunctions.identity(), false, maxConcurrency); + } + + /** + * Merges all Singles emitted by the Observable and runs them together, + * delaying errors from them and the Observable, until the source + * Observable and all inner Singles complete normally. + *
+ *
Backpressure:
+ *
The operator consumes items from the Observable in an unbounded manner and honors downstream backpressure.
+ *
Scheduler:
+ *
{@code merge} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param the value type of the inner Singles and the resulting Observable + * @param sources the Observable that emits Singles to be merged + * @return the new Observable instance + * @see #mergeDelayError(Observable, int) + * @see #merge(Observable) + * @see #merge(Observable, int) + * @since 1.2.7 - experimental + */ + @Experimental + public static Observable mergeDelayError(Observable> sources) { + return merge(sources, Integer.MAX_VALUE); + } + + /** + * Merges the Singles emitted by the Observable and runs up to the given number of them together at a time, + * delaying errors from them and the Observable, until the Observable and all inner Singles terminate. + *
+ *
Backpressure:
+ *
The operator consumes at most maxConcurrent items from the Observable and one-by-one after as the inner + * Singles terminate. The operator ignores downstream backpressure as it doesn't emit items but + * only the terminal event.
+ *
Scheduler:
+ *
{@code flatMapSingle} does not operate by default on a particular {@link Scheduler}.
+ *
+ * @param the value type of the inner Singles and the resulting Observable + * @param sources the Observable that emits Singles to be merged + * @param maxConcurrency the maximum number of inner Singles to run at a time + * @return the new Observable instance + * @since 1.2.7 - experimental + */ + @SuppressWarnings({ "unchecked", "rawtypes" }) + @Experimental + public static Observable mergeDelayError(Observable> sources, int maxConcurrency) { + return sources.flatMapSingle((Func1)UtilityFunctions.identity(), true, maxConcurrency); + } + /** * Returns a Single that emits the results of a specified combiner function applied to two items emitted by * two other Singles. diff --git a/src/main/java/rx/internal/operators/OnSubscribeFlatMapCompletable.java b/src/main/java/rx/internal/operators/OnSubscribeFlatMapCompletable.java new file mode 100644 index 0000000000..bfb8d6d69a --- /dev/null +++ b/src/main/java/rx/internal/operators/OnSubscribeFlatMapCompletable.java @@ -0,0 +1,214 @@ +/** + * Copyright 2017 Netflix, Inc. + * + * Licensed 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 rx.internal.operators; + +import java.util.concurrent.atomic.*; + +import rx.*; +import rx.exceptions.Exceptions; +import rx.functions.Func1; +import rx.internal.util.ExceptionsUtils; +import rx.plugins.RxJavaHooks; +import rx.subscriptions.CompositeSubscription; + +/** + * Maps upstream values to Completables and merges them, up to a given + * number of them concurrently, optionally delaying errors. + * @param the upstream value type + * @since 1.2.7 - experimental + */ +public final class OnSubscribeFlatMapCompletable implements Observable.OnSubscribe { + + final Observable source; + + final Func1 mapper; + + final boolean delayErrors; + + final int maxConcurrency; + + public OnSubscribeFlatMapCompletable(Observable source, Func1 mapper, + boolean delayErrors, int maxConcurrency) { + if (mapper == null) { + throw new NullPointerException("mapper is null"); + } + if (maxConcurrency <= 0) { + throw new IllegalArgumentException("maxConcurrency > 0 required but it was " + maxConcurrency); + } + this.source = source; + this.mapper = mapper; + this.delayErrors = delayErrors; + this.maxConcurrency = maxConcurrency; + } + + @Override + public void call(Subscriber child) { + FlatMapCompletableSubscriber parent = new FlatMapCompletableSubscriber(child, mapper, delayErrors, maxConcurrency); + child.add(parent); + child.add(parent.set); + source.unsafeSubscribe(parent); + } + + static final class FlatMapCompletableSubscriber extends Subscriber { + + final Subscriber actual; + + final Func1 mapper; + + final boolean delayErrors; + + final int maxConcurrency; + + final AtomicInteger wip; + + final CompositeSubscription set; + + final AtomicReference errors; + + FlatMapCompletableSubscriber(Subscriber actual, Func1 mapper, + boolean delayErrors, int maxConcurrency) { + this.actual = actual; + this.mapper = mapper; + this.delayErrors = delayErrors; + this.maxConcurrency = maxConcurrency; + this.wip = new AtomicInteger(1); + this.errors = new AtomicReference(); + this.set = new CompositeSubscription(); + this.request(maxConcurrency != Integer.MAX_VALUE ? maxConcurrency : Long.MAX_VALUE); + } + + @Override + public void onNext(T t) { + Completable c; + + try { + c = mapper.call(t); + if (c == null) { + throw new NullPointerException("The mapper returned a null Completable"); + } + } catch (Throwable ex) { + Exceptions.throwIfFatal(ex); + unsubscribe(); + onError(ex); + return; + } + + InnerSubscriber inner = new InnerSubscriber(); + set.add(inner); + wip.getAndIncrement(); + + c.unsafeSubscribe(inner); + } + + @Override + public void onError(Throwable e) { + if (delayErrors) { + ExceptionsUtils.addThrowable(errors, e); + onCompleted(); + } else { + set.unsubscribe(); + if (errors.compareAndSet(null, e)) { + actual.onError(ExceptionsUtils.terminate(errors)); + } else { + RxJavaHooks.onError(e); + } + } + } + + @Override + public void onCompleted() { + done(); + } + + boolean done() { + if (wip.decrementAndGet() == 0) { + Throwable ex = ExceptionsUtils.terminate(errors); + if (ex != null) { + actual.onError(ex); + } else { + actual.onCompleted(); + } + return true; + } + return false; + } + + public void innerError(InnerSubscriber inner, Throwable e) { + set.remove(inner); + if (delayErrors) { + ExceptionsUtils.addThrowable(errors, e); + if (!done() && maxConcurrency != Integer.MAX_VALUE) { + request(1); + } + } else { + set.unsubscribe(); + unsubscribe(); + if (errors.compareAndSet(null, e)) { + actual.onError(ExceptionsUtils.terminate(errors)); + } else { + RxJavaHooks.onError(e); + } + } + } + + public void innerComplete(InnerSubscriber inner) { + set.remove(inner); + if (!done() && maxConcurrency != Integer.MAX_VALUE) { + request(1); + } + } + + final class InnerSubscriber + extends AtomicReference + implements CompletableSubscriber, Subscription { + + private static final long serialVersionUID = -8588259593722659900L; + + @Override + public void unsubscribe() { + Subscription s = getAndSet(this); + if (s != null && s != this) { + s.unsubscribe(); + } + } + + @Override + public boolean isUnsubscribed() { + return get() == this; + } + + @Override + public void onCompleted() { + innerComplete(this); + } + + @Override + public void onError(Throwable e) { + innerError(this, e); + } + + @Override + public void onSubscribe(Subscription d) { + if (!compareAndSet(null, d)) { + d.unsubscribe(); + if (get() != this) { + RxJavaHooks.onError(new IllegalStateException("Subscription already set!")); + } + } + } + } + } +} diff --git a/src/main/java/rx/internal/operators/OnSubscribeFlatMapSingle.java b/src/main/java/rx/internal/operators/OnSubscribeFlatMapSingle.java new file mode 100644 index 0000000000..b64a4ef760 --- /dev/null +++ b/src/main/java/rx/internal/operators/OnSubscribeFlatMapSingle.java @@ -0,0 +1,333 @@ +/** + * Copyright 2017 Netflix, Inc. + * + * Licensed 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 rx.internal.operators; + +import java.util.Queue; +import java.util.concurrent.atomic.*; + +import rx.*; +import rx.exceptions.Exceptions; +import rx.functions.Func1; +import rx.internal.util.ExceptionsUtils; +import rx.internal.util.atomic.MpscLinkedAtomicQueue; +import rx.internal.util.unsafe.*; +import rx.plugins.RxJavaHooks; +import rx.subscriptions.CompositeSubscription; + +/** + * Maps upstream values to Singles and merges them, up to a given + * number of them concurrently, optionally delaying errors. + * @param the upstream value type + * @param the inner Singles and result value type + * @since 1.2.7 - experimental + */ +public final class OnSubscribeFlatMapSingle implements Observable.OnSubscribe { + + final Observable source; + + final Func1> mapper; + + final boolean delayErrors; + + final int maxConcurrency; + + public OnSubscribeFlatMapSingle(Observable source, Func1> mapper, + boolean delayErrors, int maxConcurrency) { + if (mapper == null) { + throw new NullPointerException("mapper is null"); + } + if (maxConcurrency <= 0) { + throw new IllegalArgumentException("maxConcurrency > 0 required but it was " + maxConcurrency); + } + this.source = source; + this.mapper = mapper; + this.delayErrors = delayErrors; + this.maxConcurrency = maxConcurrency; + } + + @Override + public void call(Subscriber child) { + FlatMapSingleSubscriber parent = new FlatMapSingleSubscriber(child, mapper, delayErrors, maxConcurrency); + child.add(parent.set); + child.add(parent.requested); + child.setProducer(parent.requested); + source.unsafeSubscribe(parent); + } + + static final class FlatMapSingleSubscriber extends Subscriber { + + final Subscriber actual; + + final Func1> mapper; + + final boolean delayErrors; + + final int maxConcurrency; + + final AtomicInteger wip; + + final AtomicInteger active; + + final CompositeSubscription set; + + final AtomicReference errors; + + final Queue queue; + + final Requested requested; + + volatile boolean done; + + volatile boolean cancelled; + + FlatMapSingleSubscriber(Subscriber actual, + Func1> mapper, + boolean delayErrors, int maxConcurrency) { + this.actual = actual; + this.mapper = mapper; + this.delayErrors = delayErrors; + this.maxConcurrency = maxConcurrency; + this.wip = new AtomicInteger(); + this.errors = new AtomicReference(); + this.requested = new Requested(); + this.set = new CompositeSubscription(); + this.active = new AtomicInteger(); + if (UnsafeAccess.isUnsafeAvailable()) { + queue = new MpscLinkedQueue(); + } else { + queue = new MpscLinkedAtomicQueue(); + } + this.request(maxConcurrency != Integer.MAX_VALUE ? maxConcurrency : Long.MAX_VALUE); + } + + @Override + public void onNext(T t) { + Single c; + + try { + c = mapper.call(t); + if (c == null) { + throw new NullPointerException("The mapper returned a null Single"); + } + } catch (Throwable ex) { + Exceptions.throwIfFatal(ex); + unsubscribe(); + onError(ex); + return; + } + + InnerSubscriber inner = new InnerSubscriber(); + set.add(inner); + active.incrementAndGet(); + + c.subscribe(inner); + } + + @Override + public void onError(Throwable e) { + if (delayErrors) { + ExceptionsUtils.addThrowable(errors, e); + } else { + set.unsubscribe(); + if (!errors.compareAndSet(null, e)) { + RxJavaHooks.onError(e); + return; + } + } + done = true; + drain(); + } + + @Override + public void onCompleted() { + done = true; + drain(); + } + + void innerSuccess(InnerSubscriber inner, R value) { + queue.offer(NotificationLite.next(value)); + set.remove(inner); + active.decrementAndGet(); + drain(); + } + + void innerError(InnerSubscriber inner, Throwable e) { + if (delayErrors) { + ExceptionsUtils.addThrowable(errors, e); + set.remove(inner); + if (!done && maxConcurrency != Integer.MAX_VALUE) { + request(1); + } + } else { + set.unsubscribe(); + unsubscribe(); + if (!errors.compareAndSet(null, e)) { + RxJavaHooks.onError(e); + return; + } + done = true; + } + active.decrementAndGet(); + drain(); + } + + void drain() { + if (wip.getAndIncrement() != 0) { + return; + } + + int missed = 1; + Subscriber a = actual; + Queue q = queue; + boolean delayError = this.delayErrors; + AtomicInteger act = active; + + for (;;) { + long r = requested.get(); + long e = 0L; + + while (e != r) { + if (cancelled) { + q.clear(); + return; + } + + boolean d = done; + + if (!delayError && d) { + Throwable ex = errors.get(); + if (ex != null) { + q.clear(); + a.onError(ExceptionsUtils.terminate(errors)); + return; + } + } + + Object o = q.poll(); + + boolean empty = o == null; + + if (d && act.get() == 0 && empty) { + Throwable ex = errors.get(); + if (ex != null) { + a.onError(ExceptionsUtils.terminate(errors)); + } else { + a.onCompleted(); + } + return; + } + + if (empty) { + break; + } + + a.onNext(NotificationLite.getValue(o)); + + e++; + } + + if (e == r) { + if (cancelled) { + q.clear(); + return; + } + + if (done) { + if (delayError) { + if (act.get() == 0 && q.isEmpty()) { + Throwable ex = errors.get(); + if (ex != null) { + a.onError(ExceptionsUtils.terminate(errors)); + } else { + a.onCompleted(); + } + return; + } + } else { + Throwable ex = errors.get(); + if (ex != null) { + q.clear(); + a.onError(ExceptionsUtils.terminate(errors)); + return; + } + else if (act.get() == 0 && q.isEmpty()) { + a.onCompleted(); + return; + } + } + } + } + + if (e != 0L) { + requested.produced(e); + if (!done && maxConcurrency != Integer.MAX_VALUE) { + request(e); + } + } + + missed = wip.addAndGet(-missed); + if (missed == 0) { + break; + } + } + } + + final class Requested extends AtomicLong implements Producer, Subscription { + + private static final long serialVersionUID = -887187595446742742L; + + @Override + public void request(long n) { + if (n > 0L) { + BackpressureUtils.getAndAddRequest(this, n); + drain(); + } + } + + void produced(long e) { + BackpressureUtils.produced(this, e); + } + + @Override + public void unsubscribe() { + cancelled = true; + FlatMapSingleSubscriber.this.unsubscribe(); + if (wip.getAndIncrement() == 0) { + queue.clear(); + } + } + + @Override + public boolean isUnsubscribed() { + return cancelled; + } + } + + final class InnerSubscriber extends SingleSubscriber { + + @Override + public void onSuccess(R t) { + innerSuccess(this, t); + } + + @Override + public void onError(Throwable error) { + innerError(this, error); + } + } + } +} diff --git a/src/test/java/rx/CompletableTest.java b/src/test/java/rx/CompletableTest.java index 23418b60e6..3f34a983ca 100644 --- a/src/test/java/rx/CompletableTest.java +++ b/src/test/java/rx/CompletableTest.java @@ -93,7 +93,7 @@ public void remove() { /** * A class containing a completable instance and counts the number of subscribers. */ - static final class NormalCompletable extends AtomicInteger { + public static final class NormalCompletable extends AtomicInteger { /** */ private static final long serialVersionUID = 7192337844700923752L; @@ -119,7 +119,7 @@ public void assertSubscriptions(int n) { * A class containing a completable instance that emits a TestException and counts * the number of subscribers. */ - static final class ErrorCompletable extends AtomicInteger { + public static final class ErrorCompletable extends AtomicInteger { /** */ private static final long serialVersionUID = 7192337844700923752L; diff --git a/src/test/java/rx/internal/operators/OnSubscribeFlatMapCompletableTest.java b/src/test/java/rx/internal/operators/OnSubscribeFlatMapCompletableTest.java new file mode 100644 index 0000000000..2b88387400 --- /dev/null +++ b/src/test/java/rx/internal/operators/OnSubscribeFlatMapCompletableTest.java @@ -0,0 +1,557 @@ +/** + * Copyright 2017 Netflix, Inc. + * + * Licensed 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 rx.internal.operators; + +import static org.junit.Assert.*; + +import java.util.*; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicInteger; + +import org.junit.*; + +import rx.*; +import rx.CompletableTest.*; +import rx.Observable; +import rx.exceptions.*; +import rx.functions.*; +import rx.internal.util.UtilityFunctions; +import rx.observers.*; +import rx.schedulers.Schedulers; +import rx.subjects.PublishSubject; +import rx.subscriptions.Subscriptions; + +public class OnSubscribeFlatMapCompletableTest implements Action0, Action1 { + + final AtomicInteger calls = new AtomicInteger(); + + /** A normal Completable object. */ + final NormalCompletable normal = new NormalCompletable(); + + /** An error Completable object. */ + final ErrorCompletable error = new ErrorCompletable(); + + final Func1 identity = UtilityFunctions.identity(); + + @Override + public void call() { + calls.getAndIncrement(); + } + + @Override + public void call(Object t) { + calls.getAndIncrement(); + } + + void assertCalls(int n) { + assertEquals(n, calls.get()); + } + + @Test + public void normal() { + Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.complete().doOnCompleted(OnSubscribeFlatMapCompletableTest.this); + } + }) + .test() + .assertResult(); + + assertCalls(10); + } + + @Test + public void normalMaxConcurrent() { + for (int i = 1; i < 10; i++) { + calls.set(0); + + Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.complete() + .observeOn(Schedulers.computation()) + .doOnCompleted(OnSubscribeFlatMapCompletableTest.this); + } + }, false, i) + .test() + .awaitTerminalEvent(5, TimeUnit.SECONDS) + .assertResult(); + + assertCalls(10); + } + } + + @Test + public void error() { + Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.error(new TestException()).doOnError(OnSubscribeFlatMapCompletableTest.this); + } + }) + .test() + .assertFailure(TestException.class); + + assertCalls(1); + } + + @Test + public void errorDelayed() { + AssertableSubscriber as = Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.error(new TestException()).doOnError(OnSubscribeFlatMapCompletableTest.this); + } + }, true) + .test() + .assertFailure(CompositeException.class); + + List onErrorEvents = as.getOnErrorEvents(); + + assertEquals(onErrorEvents.toString(), 1, onErrorEvents.size()); + + onErrorEvents = ((CompositeException)onErrorEvents.get(0)).getExceptions(); + + assertEquals(onErrorEvents.toString(), 10, onErrorEvents.size()); + + for (Throwable ex : onErrorEvents) { + assertTrue(ex.toString(), ex instanceof TestException); + } + + assertCalls(10); + } + + @Test + public void errorDelayedMaxConcurrency() { + AssertableSubscriber as = Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.error(new TestException()).doOnError(OnSubscribeFlatMapCompletableTest.this); + } + }, true, 1) + .test() + .assertFailure(CompositeException.class); + + List onErrorEvents = as.getOnErrorEvents(); + + assertEquals(onErrorEvents.toString(), 1, onErrorEvents.size()); + + onErrorEvents = ((CompositeException)onErrorEvents.get(0)).getExceptions(); + + assertEquals(onErrorEvents.toString(), 10, onErrorEvents.size()); + + for (Throwable ex : onErrorEvents) { + assertTrue(ex.toString(), ex instanceof TestException); + } + + assertCalls(10); + } + + @Test + public void mapperThrows() { + Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + throw new TestException(); + } + }) + .test() + .assertFailure(TestException.class); + } + + @Test + public void mapperNull() { + Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return null; + } + }) + .test() + .assertFailure(NullPointerException.class); + } + + @Test + public void paramValidation() { + try { + Observable.range(1, 10) + .flatMapCompletable(null); + fail("Should have thrown"); + } catch (NullPointerException ex) { + assertEquals("mapper is null", ex.getMessage()); + } + + try { + Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.complete(); + } + }, false, 0); + fail("Should have thrown"); + } catch (IllegalArgumentException ex) { + assertEquals("maxConcurrency > 0 required but it was 0", ex.getMessage()); + } + + try { + Observable.range(1, 10) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.complete(); + } + }, true, -99); + fail("Should have thrown"); + } catch (IllegalArgumentException ex) { + assertEquals("maxConcurrency > 0 required but it was -99", ex.getMessage()); + } + } + + @Test + public void mainErrorDelayed() { + Observable.range(1, 10).concatWith(Observable.error(new TestException())) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return Completable.complete().doOnCompleted(OnSubscribeFlatMapCompletableTest.this); + } + }, true) + .test() + .assertFailure(TestException.class); + + assertCalls(10); + } + + @Test + public void innerDoubleOnSubscribe() { + final CompletableSubscriber[] inner = { null }; + + AssertableSubscriber as = Observable.just(1) + .flatMapCompletable(new Func1() { + @Override + public Completable call(Integer t) { + return Completable.create(new Completable.OnSubscribe() { + @Override + public void call(CompletableSubscriber t) { + OnSubscribeFlatMapCompletableTest.this.call(); + Subscription s1 = Subscriptions.empty(); + + t.onSubscribe(s1); + + Subscription s2 = Subscriptions.empty(); + + t.onSubscribe(s2); + + if (s2.isUnsubscribed()) { + OnSubscribeFlatMapCompletableTest.this.call(); + } + + t.onCompleted(); + + inner[0] = t; + } + }); + } + }) + .test() + .assertResult(); + + assertCalls(2); + + inner[0].onError(new TestException()); + + as.assertResult(); + } + + @Test + public void mainErrorUnsubscribes() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(); + + ps0.flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return v == 0 ? ps1.toCompletable() : ps2.toCompletable(); + } + }).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onError(new TestException()); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + + @Test + public void innerErrorUnsubscribes() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(); + + ps0.flatMapCompletable(new Func1() { + @Override + public Completable call(Integer v) { + return v == 0 ? ps1.toCompletable() : ps2.toCompletable(); + } + }).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps1.onError(new TestException()); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + + + @Test(timeout = 5000) + public void mergeObservableEmpty() { + Completable c = Observable.empty().flatMapCompletable(identity).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000, expected = TestException.class) + public void mergeObservableError() { + Completable c = Observable.error(new TestException()).flatMapCompletable(identity).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000) + public void mergeObservableSingle() { + Completable c = Observable.just(normal.completable).flatMapCompletable(identity).toCompletable(); + + c.await(); + + normal.assertSubscriptions(1); + } + + @Test(timeout = 5000, expected = TestException.class) + public void mergeObservableSingleThrows() { + Completable c = Observable.just(error.completable).flatMapCompletable(identity).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000) + public void mergeObservableMany() { + Completable c = Observable.just(normal.completable).repeat(3).flatMapCompletable(identity).toCompletable(); + + c.await(); + + normal.assertSubscriptions(3); + } + + @Test(timeout = 5000, expected = TestException.class) + public void mergeObservableManyOneThrows() { + Completable c = Observable.just(normal.completable, error.completable).flatMapCompletable(identity).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000) + public void mergeObservableMaxConcurrent() { + final List requested = new ArrayList(); + Observable cs = Observable + .just(normal.completable) + .repeat(10) + .doOnRequest(new Action1() { + @Override + public void call(Long v) { + requested.add(v); + } + }); + + Completable c = cs.flatMapCompletable(identity, false, 5).toCompletable(); + + c.await(); + + // FIXME this request pattern looks odd because all 10 completions trigger 1 requests + Assert.assertEquals(Arrays.asList(5L, 1L, 1L, 1L, 1L, 1L, 1L, 1L, 1L, 1L, 1L), requested); + } + + @Test(timeout = 5000) + public void mergeDelayErrorObservableEmpty() { + Completable c = Observable.empty().flatMapCompletable(identity, true).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000, expected = TestException.class) + public void mergeDelayErrorObservableError() { + Completable c = Observable.error(new TestException()).flatMapCompletable(identity, true).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000) + public void mergeDelayErrorObservableSingle() { + Completable c = Observable.just(normal.completable).flatMapCompletable(identity, true).toCompletable(); + + c.await(); + + normal.assertSubscriptions(1); + } + + @Test(timeout = 5000, expected = TestException.class) + public void mergeDelayErrorObservableSingleThrows() { + Completable c = Observable.just(error.completable).flatMapCompletable(identity, true).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000) + public void mergeDelayErrorObservableMany() { + Completable c = Observable.just(normal.completable).repeat(3).flatMapCompletable(identity, true).toCompletable(); + + c.await(); + + normal.assertSubscriptions(3); + } + + @Test(timeout = 5000, expected = TestException.class) + public void mergeDelayErrorObservableManyOneThrows() { + Completable c = Observable.just(normal.completable, error.completable).flatMapCompletable(identity, true).toCompletable(); + + c.await(); + } + + @Test(timeout = 5000) + public void mergeDelayErrorObservableMaxConcurrent() { + final List requested = new ArrayList(); + Observable cs = Observable + .just(normal.completable) + .repeat(10) + .doOnRequest(new Action1() { + @Override + public void call(Long v) { + requested.add(v); + } + }); + + Completable c = cs.flatMapCompletable(identity, true, 5).toCompletable(); + + c.await(); + + // FIXME this request pattern looks odd because all 10 completions trigger 1 requests + Assert.assertEquals(Arrays.asList(5L, 1L, 1L, 1L, 1L, 1L, 1L, 1L, 1L, 1L, 1L), requested); + } + + @Test + public void asyncObservables() { + + final int[] calls = { 0 }; + + Observable.range(1, 5).map(new Func1() { + @Override + public Completable call(final Integer v) { + System.out.println("Mapping " + v); + return Completable.fromAction(new Action0() { + @Override + public void call() { + System.out.println("Processing " + (calls[0] + 1)); + calls[0]++; + } + }) + .subscribeOn(Schedulers.io()) + .doOnCompleted(new Action0() { + @Override + public void call() { + System.out.println("Inner complete " + v); + } + }) + .observeOn(Schedulers.computation()); + } + }).flatMapCompletable(identity, false, 1).toCompletable() + .test() + .awaitTerminalEventAndUnsubscribeOnTimeout(5, TimeUnit.SECONDS) + .assertResult(); + + Assert.assertEquals(5, calls[0]); + } + +} diff --git a/src/test/java/rx/internal/operators/OnSubscribeFlatMapSingleTest.java b/src/test/java/rx/internal/operators/OnSubscribeFlatMapSingleTest.java new file mode 100644 index 0000000000..3da2ac389c --- /dev/null +++ b/src/test/java/rx/internal/operators/OnSubscribeFlatMapSingleTest.java @@ -0,0 +1,782 @@ +/** + * Copyright 2017 Netflix, Inc. + * + * Licensed 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 rx.internal.operators; + +import java.util.*; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicInteger; + +import static org.junit.Assert.*; +import org.junit.Test; + +import rx.*; +import rx.Observable; +import rx.exceptions.*; +import rx.functions.*; +import rx.observers.*; +import rx.schedulers.Schedulers; +import rx.subjects.PublishSubject; + +public class OnSubscribeFlatMapSingleTest implements Action0, Action1 { + + final AtomicInteger calls = new AtomicInteger(); + + final Action1 errorConsumer = new Action1() { + @Override + public void call(Throwable e) { + OnSubscribeFlatMapSingleTest.this.call(e); + } + }; + + @Override + public void call() { + calls.getAndIncrement(); + } + + @Override + public void call(Object t) { + calls.getAndIncrement(); + } + + void assertCalls(int n) { + assertEquals(n, calls.get()); + } + + @Test + public void normal() { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }) + .test() + .assertResult(1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + } + + @Test + public void normalBackpressured() { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }) + .test(0L) + .assertNoValues() + .requestMore(1) + .assertValues(1) + .requestMore(2) + .assertValues(1, 2, 3) + .requestMore(3) + .assertValues(1, 2, 3, 4, 5, 6) + .requestMore(4) + .assertResult(1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + } + + @Test + public void normalMaxConcurrencyBackpressured() { + for (int i = 1; i < 16; i++) { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }, false, i) + .test(0L) + .assertNoValues() + .requestMore(1) + .assertValues(1) + .requestMore(2) + .assertValues(1, 2, 3) + .requestMore(3) + .assertValues(1, 2, 3, 4, 5, 6) + .requestMore(4) + .assertResult(1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + } + } + + @Test + public void normalMaxConcurrent() { + for (int i = 1; i < 16; i++) { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }, false, i) + .test() + .assertResult(1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + } + } + + @Test + public void normalMaxConcurrentAsync() { + for (int i = 1; i < 2; i++) { + AssertableSubscriber as = Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v).observeOn(Schedulers.computation()); + } + }, false, i) + .test(); + + as.awaitTerminalEvent(5, TimeUnit.SECONDS) + .assertValueCount(10) + .assertNoErrors() + .assertCompleted(); + + Set set = new HashSet(as.getOnNextEvents()); + + for (int j = 1; j < 11; j++) { + assertTrue("" + set, set.contains(j)); + } + } + } + + @Test + public void justMaxConcurrentAsync() { + AssertableSubscriber as = Observable.just(1) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v).observeOn(Schedulers.computation()); + } + }, false, 1) + .test(); + + as.awaitTerminalEvent(5, TimeUnit.SECONDS) + .assertResult(1); + } + + @Test + public void error() { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.error(new TestException()).doOnError(errorConsumer); + } + }) + .test() + .assertFailure(TestException.class); + + assertCalls(1); + } + + @Test + public void errorDelayed() { + AssertableSubscriber as = Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.error(new TestException()).doOnError(errorConsumer); + } + }, true) + .test() + .assertFailure(CompositeException.class); + + List onErrorEvents = as.getOnErrorEvents(); + + assertEquals(onErrorEvents.toString(), 1, onErrorEvents.size()); + + onErrorEvents = ((CompositeException)onErrorEvents.get(0)).getExceptions(); + + assertEquals(onErrorEvents.toString(), 10, onErrorEvents.size()); + + for (Throwable ex : onErrorEvents) { + assertTrue(ex.toString(), ex instanceof TestException); + } + + assertCalls(10); + } + + @Test + public void errorDelayedMaxConcurrency() { + AssertableSubscriber as = Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.error(new TestException()).doOnError(errorConsumer); + } + }, true, 1) + .test() + .assertFailure(CompositeException.class); + + List onErrorEvents = as.getOnErrorEvents(); + + assertEquals(onErrorEvents.toString(), 1, onErrorEvents.size()); + + onErrorEvents = ((CompositeException)onErrorEvents.get(0)).getExceptions(); + + assertEquals(onErrorEvents.toString(), 10, onErrorEvents.size()); + + for (Throwable ex : onErrorEvents) { + assertTrue(ex.toString(), ex instanceof TestException); + } + + assertCalls(10); + } + + @Test + public void mapperThrows() { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + throw new TestException(); + } + }) + .test() + .assertFailure(TestException.class); + } + + @Test + public void mapperNull() { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return null; + } + }) + .test() + .assertFailure(NullPointerException.class); + } + + @Test + public void paramValidation() { + try { + Observable.range(1, 10) + .flatMapSingle(null); + fail("Should have thrown"); + } catch (NullPointerException ex) { + assertEquals("mapper is null", ex.getMessage()); + } + + try { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }, false, 0); + fail("Should have thrown"); + } catch (IllegalArgumentException ex) { + assertEquals("maxConcurrency > 0 required but it was 0", ex.getMessage()); + } + + try { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }, true, -99); + fail("Should have thrown"); + } catch (IllegalArgumentException ex) { + assertEquals("maxConcurrency > 0 required but it was -99", ex.getMessage()); + } + } + + @Test + public void mainErrorDelayed() { + Observable.range(1, 10).concatWith(Observable.error(new TestException())) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v).doOnSuccess(OnSubscribeFlatMapSingleTest.this); + } + }, true) + .test() + .assertFailure(TestException.class, 1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + + assertCalls(10); + } + + @Test + public void mainErrorUnsubscribes() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(); + + ps0.flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return v == 0 ? ps1.toSingle() : ps2.toSingle(); + } + }).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onError(new TestException()); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + + @Test + public void innerErrorUnsubscribes() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(); + + ps0.flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return v == 0 ? ps1.toSingle() : ps2.toSingle(); + } + }).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps1.onError(new TestException()); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + + @Test + public void take() { + Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }) + .take(5) + .test() + .assertResult(1, 2, 3, 4, 5); + } + + + @Test + public void unsubscribe() { + AssertableSubscriber as = Observable.range(1, 10) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }) + .test(0) + ; + + as.unsubscribe(); + + as.assertNoValues().assertNoErrors().assertNotCompleted(); + } + + @Test + public void mainErrorUnsubscribesNoRequest() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(0L); + + ps0.flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return v == 0 ? ps1.toSingle() : ps2.toSingle(); + } + }).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onError(new TestException()); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + + @Test + public void innerErrorUnsubscribesNoRequest() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(0L); + + ps0.flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return v == 0 ? ps1.toSingle() : ps2.toSingle(); + } + }).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps1.onError(new TestException()); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + @Test + public void mainErrorUnsubscribesNoRequestDelayError() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(0L); + + ps0.flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return v == 0 ? ps1.toSingle() : ps2.toSingle(); + } + }, true).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onError(new TestException()); + ps1.onNext(3); + ps1.onCompleted(); + ps2.onNext(4); + ps2.onCompleted(); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + + as.requestMore(2); + as.assertValues(3, 4); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + + @Test + public void innerErrorUnsubscribesNoRequestDelayError() { + PublishSubject ps0 = PublishSubject.create(); + final PublishSubject ps1 = PublishSubject.create(); + final PublishSubject ps2 = PublishSubject.create(); + + TestSubscriber as = TestSubscriber.create(0L); + + ps0.flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return v == 0 ? ps1.toSingle() : ps2.toSingle(); + } + }, true).unsafeSubscribe(as); + + assertTrue(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(0); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onNext(1); + + assertTrue(ps0.hasObservers()); + assertTrue(ps1.hasObservers()); + assertTrue(ps2.hasObservers()); + as.assertNoValues(); + as.assertNoErrors(); + as.assertNotCompleted(); + + ps0.onCompleted(); + ps1.onError(new TestException()); + ps2.onNext(4); + ps2.onCompleted(); + + assertFalse(ps0.hasObservers()); + assertFalse(ps1.hasObservers()); + assertFalse(ps2.hasObservers()); + + as.requestMore(1); + as.assertValues(4); + as.assertError(TestException.class); + as.assertNotCompleted(); + } + + @Test + public void justBackpressured() { + Observable.just(1) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }) + .test(1L) + .assertResult(1); + } + + @Test + public void justBackpressuredDelayError() { + Observable.just(1) + .flatMapSingle(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + }, true) + .test(1L) + .assertResult(1); + } + + @Test + public void singleMerge() { + Single.merge(Observable.range(1, 10).map(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + })) + .test() + .assertResult(1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + } + + @Test + public void singleMergeMaxConcurrent() { + AssertableSubscriber as = Single.merge(Observable.range(1, 10).map(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v).observeOn(Schedulers.computation()); + } + }), 2) + .test() + .awaitTerminalEvent(5, TimeUnit.SECONDS); + + Set set = new HashSet(as.getOnNextEvents()); + + assertEquals("" + set, 10, set.size()); + for (int j = 1; j < 11; j++) { + assertTrue("" + set, set.contains(j)); + } + } + + @Test + public void singleMergeDelayError() { + Single.mergeDelayError(Observable.range(1, 10).map(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + })) + .test() + .assertResult(1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + } + + @Test + public void singleMergeDelayErrorMaxConcurrent() { + AssertableSubscriber as = Single.mergeDelayError(Observable.range(1, 10).map(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v).observeOn(Schedulers.computation()); + } + }), 2) + .test() + .awaitTerminalEvent(5, TimeUnit.SECONDS); + + Set set = new HashSet(as.getOnNextEvents()); + + assertEquals("" + set, 10, set.size()); + for (int j = 1; j < 11; j++) { + assertTrue("" + set, set.contains(j)); + } + } + + @Test + public void singleMergeDelayErrorWithError() { + Single.mergeDelayError(Observable.range(1, 10) + .concatWith(Observable.error(new TestException())) + .map(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v); + } + })) + .test() + .assertFailure(TestException.class, 1, 2, 3, 4, 5, 6, 7, 8, 9, 10); + } + + @Test + public void singleMergeDelayMaxConcurrentErrorWithError() { + AssertableSubscriber as = Single.mergeDelayError(Observable.range(1, 10) + .concatWith(Observable.error(new TestException())) + .map(new Func1>() { + @Override + public Single call(Integer v) { + return Single.just(v).observeOn(Schedulers.computation()); + } + }), 2) + .test() + .awaitTerminalEvent(5, TimeUnit.SECONDS); + + Set set = new HashSet(as.getOnNextEvents()); + + assertEquals("" + set, 10, set.size()); + for (int j = 1; j < 11; j++) { + assertTrue("" + set, set.contains(j)); + } + } +}